Repository: spark
Updated Branches:
  refs/heads/master aca2a0165 -> 5f5698012


[SPARK-11137][STREAMING] Make StreamingContext.stop() exception-safe

Make StreamingContext.stop() exception-safe

Author: jayadevanmurali <jayadeva...@tcs.com>

Closes #10807 from jayadevanmurali/branch-0.1-SPARK-11137.


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

Branch: refs/heads/master
Commit: 5f56980127704d3c2877d0d0b5047791c00fdac9
Parents: aca2a01
Author: jayadevanmurali <jayadeva...@tcs.com>
Authored: Sat Jan 23 11:48:48 2016 +0000
Committer: Sean Owen <so...@cloudera.com>
Committed: Sat Jan 23 11:48:48 2016 +0000

----------------------------------------------------------------------
 .../apache/spark/streaming/StreamingContext.scala   | 16 ++++++++++++----
 1 file changed, 12 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/spark/blob/5f569801/streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala
----------------------------------------------------------------------
diff --git 
a/streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala 
b/streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala
index ec57c05..32bea88 100644
--- a/streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala
+++ b/streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala
@@ -693,12 +693,20 @@ class StreamingContext private[streaming] (
           // interrupted. See SPARK-12001 for more details. Because the body 
of this case can be
           // executed twice in the case of a partial stop, all methods called 
here need to be
           // idempotent.
-          scheduler.stop(stopGracefully)
+          Utils.tryLogNonFatalError {
+            scheduler.stop(stopGracefully)
+          }
           // Removing the streamingSource to de-register the metrics on stop()
-          env.metricsSystem.removeSource(streamingSource)
-          uiTab.foreach(_.detach())
+          Utils.tryLogNonFatalError {
+            env.metricsSystem.removeSource(streamingSource)
+          }
+          Utils.tryLogNonFatalError {
+            uiTab.foreach(_.detach())
+          }
           StreamingContext.setActiveContext(null)
-          waiter.notifyStop()
+          Utils.tryLogNonFatalError {
+            waiter.notifyStop()
+          }
           if (shutdownHookRef != null) {
             shutdownHookRefToRemove = shutdownHookRef
             shutdownHookRef = null


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

Reply via email to