From b566b66dbf6dd380512f11fcee2ef77d0461522b Mon Sep 17 00:00:00 2001 From: Joshi Date: Tue, 23 Jun 2015 21:42:29 -0700 Subject: [PATCH] SPARK-2645: Fix for SparkContext stop behavior --- .../scala/org/apache/spark/SparkEnv.scala | 28 ++++++++++++------- 1 file changed, 18 insertions(+), 10 deletions(-) diff --git a/core/src/main/scala/org/apache/spark/SparkEnv.scala b/core/src/main/scala/org/apache/spark/SparkEnv.scala index b0665570e2681..5b05a17bcbe84 100644 --- a/core/src/main/scala/org/apache/spark/SparkEnv.scala +++ b/core/src/main/scala/org/apache/spark/SparkEnv.scala @@ -90,17 +90,25 @@ class SparkEnv ( private var driverTmpDirToDelete: Option[String] = None private[spark] def stop() { + + if(isStopped) return + isStopped = true - pythonWorkers.foreach { case(key, worker) => worker.stop() } - Option(httpFileServer).foreach(_.stop()) - mapOutputTracker.stop() - shuffleManager.stop() - broadcastManager.stop() - blockManager.stop() - blockManager.master.stop() - metricsSystem.stop() - outputCommitCoordinator.stop() - rpcEnv.shutdown() + try { + pythonWorkers.foreach { case (key, worker) => worker.stop()} + Option(httpFileServer).foreach(_.stop()) + mapOutputTracker.stop() + shuffleManager.stop() + broadcastManager.stop() + blockManager.stop() + blockManager.master.stop() + metricsSystem.stop() + outputCommitCoordinator.stop() + rpcEnv.shutdown() + } catch { + case e: Exception => + logInfo("Exception while SparkEnv stop", e) + } // Unfortunately Akka's awaitTermination doesn't actually wait for the Netty server to shut // down, but let's call it anyway in case it gets fixed in a later release