Repository: spark
Updated Branches:
  refs/heads/branch-1.4 5118abb4e -> 94789f374


[SPARK-11870][STREAMING][PYSPARK] Rethrow the exceptions in TransformFunction 
and TransformFunctionSerializer

TransformFunction and TransformFunctionSerializer don't rethrow the exception, 
so when any exception happens, it just return None. This will cause some weird 
NPE and confuse people.

Author: Shixiong Zhu <shixi...@databricks.com>

Closes #9847 from zsxwing/pyspark-streaming-exception.

(cherry picked from commit be7a2cfd978143f6f265eca63e9e24f755bc9f22)
Signed-off-by: Tathagata Das <tathagata.das1...@gmail.com>


Project: http://git-wip-us.apache.org/repos/asf/spark/repo
Commit: http://git-wip-us.apache.org/repos/asf/spark/commit/94789f37
Tree: http://git-wip-us.apache.org/repos/asf/spark/tree/94789f37
Diff: http://git-wip-us.apache.org/repos/asf/spark/diff/94789f37

Branch: refs/heads/branch-1.4
Commit: 94789f37400ea534e051d1df19f3a567646979fd
Parents: 5118abb
Author: Shixiong Zhu <shixi...@databricks.com>
Authored: Fri Nov 20 14:23:01 2015 -0800
Committer: Tathagata Das <tathagata.das1...@gmail.com>
Committed: Fri Nov 20 14:23:57 2015 -0800

----------------------------------------------------------------------
 python/pyspark/streaming/tests.py | 16 ++++++++++++++++
 python/pyspark/streaming/util.py  |  3 +++
 2 files changed, 19 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/spark/blob/94789f37/python/pyspark/streaming/tests.py
----------------------------------------------------------------------
diff --git a/python/pyspark/streaming/tests.py 
b/python/pyspark/streaming/tests.py
index ab1cc3f..830fd9e 100644
--- a/python/pyspark/streaming/tests.py
+++ b/python/pyspark/streaming/tests.py
@@ -376,6 +376,22 @@ class BasicOperationTests(PySparkStreamingTestCase):
         expected = [[('k', v)] for v in expected]
         self._test_func(input, func, expected)
 
+    def test_failed_func(self):
+        input = [self.sc.parallelize([d], 1) for d in range(4)]
+        input_stream = self.ssc.queueStream(input)
+
+        def failed_func(i):
+            raise ValueError("failed")
+
+        input_stream.map(failed_func).pprint()
+        self.ssc.start()
+        try:
+            self.ssc.awaitTerminationOrTimeout(10)
+        except:
+            return
+
+        self.fail("a failed func should throw an error")
+
 
 class WindowFunctionTests(PySparkStreamingTestCase):
 

http://git-wip-us.apache.org/repos/asf/spark/blob/94789f37/python/pyspark/streaming/util.py
----------------------------------------------------------------------
diff --git a/python/pyspark/streaming/util.py b/python/pyspark/streaming/util.py
index 34291f3..0a2773b 100644
--- a/python/pyspark/streaming/util.py
+++ b/python/pyspark/streaming/util.py
@@ -59,6 +59,7 @@ class TransformFunction(object):
                 return r._jrdd
         except Exception:
             traceback.print_exc()
+            raise
 
     def __repr__(self):
         return "TransformFunction(%s)" % self.func
@@ -90,6 +91,7 @@ class TransformFunctionSerializer(object):
             return bytearray(self.serializer.dumps((func.func, 
func.deserializers)))
         except Exception:
             traceback.print_exc()
+            raise
 
     def loads(self, data):
         try:
@@ -97,6 +99,7 @@ class TransformFunctionSerializer(object):
             return TransformFunction(self.ctx, f, *deserializers)
         except Exception:
             traceback.print_exc()
+            raise
 
     def __repr__(self):
         return "TransformFunctionSerializer(%s)" % self.serializer


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@spark.apache.org
For additional commands, e-mail: commits-h...@spark.apache.org

Reply via email to