diff options
author | Josh Rosen <joshrosen@databricks.com> | 2016-03-14 12:22:02 -0700 |
---|---|---|
committer | Josh Rosen <joshrosen@databricks.com> | 2016-03-14 12:22:02 -0700 |
commit | 07cb323e7a128b87ef265ddc66f033365d9de463 (patch) | |
tree | 41de1d7201b8592ffd8742aa09a98ee1beb207e6 /python/pyspark/streaming/kafka.py | |
parent | 6a4bfcd62b7effcfbb865bdd301d41a0ba6e5c94 (diff) | |
download | spark-07cb323e7a128b87ef265ddc66f033365d9de463.tar.gz spark-07cb323e7a128b87ef265ddc66f033365d9de463.tar.bz2 spark-07cb323e7a128b87ef265ddc66f033365d9de463.zip |
[SPARK-13848][SPARK-5185] Update to Py4J 0.9.2 in order to fix classloading issue
This patch upgrades Py4J from 0.9.1 to 0.9.2 in order to include a patch which modifies Py4J to use the current thread's ContextClassLoader when performing reflection / class loading. This is necessary in order to fix [SPARK-5185](https://issues.apache.org/jira/browse/SPARK-5185), a longstanding issue affecting the use of `--jars` and `--packages` in PySpark.
In order to demonstrate that the fix works, I removed the workarounds which were added as part of [SPARK-6027](https://issues.apache.org/jira/browse/SPARK-6027) / #4779 and other patches.
Py4J diff: https://github.com/bartdag/py4j/compare/0.9.1...0.9.2
/cc zsxwing tdas davies brkyvz
Author: Josh Rosen <joshrosen@databricks.com>
Closes #11687 from JoshRosen/py4j-0.9.2.
Diffstat (limited to 'python/pyspark/streaming/kafka.py')
-rw-r--r-- | python/pyspark/streaming/kafka.py | 10 |
1 files changed, 3 insertions, 7 deletions
diff --git a/python/pyspark/streaming/kafka.py b/python/pyspark/streaming/kafka.py index a70b99249d..02a88699a2 100644 --- a/python/pyspark/streaming/kafka.py +++ b/python/pyspark/streaming/kafka.py @@ -192,13 +192,9 @@ class KafkaUtils(object): @staticmethod def _get_helper(sc): try: - # Use KafkaUtilsPythonHelper to access Scala's KafkaUtils (see SPARK-6027) - helperClass = sc._jvm.java.lang.Thread.currentThread().getContextClassLoader() \ - .loadClass("org.apache.spark.streaming.kafka.KafkaUtilsPythonHelper") - return helperClass.newInstance() - except Py4JJavaError as e: - # TODO: use --jar once it also work on driver - if 'ClassNotFoundException' in str(e.java_exception): + return sc._jvm.org.apache.spark.streaming.kafka.KafkaUtilsPythonHelper() + except TypeError as e: + if str(e) == "'JavaPackage' object is not callable": KafkaUtils._printErrorMsg(sc) raise |