aboutsummaryrefslogtreecommitdiff
path: root/core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala
diff options
context:
space:
mode:
Diffstat (limited to 'core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala')
-rw-r--r--core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala18
1 files changed, 6 insertions, 12 deletions
diff --git a/core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala b/core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala
index e8a1e35c3f..7fc96e4f76 100644
--- a/core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala
+++ b/core/src/main/scala/org/apache/spark/deploy/ExternalShuffleService.scala
@@ -28,7 +28,7 @@ import org.apache.spark.network.sasl.SaslServerBootstrap
import org.apache.spark.network.server.{TransportServerBootstrap, TransportServer}
import org.apache.spark.network.shuffle.ExternalShuffleBlockHandler
import org.apache.spark.network.util.TransportConf
-import org.apache.spark.util.Utils
+import org.apache.spark.util.{ShutdownHookManager, Utils}
/**
* Provides a server from which Executors can read shuffle files (rather than reading directly from
@@ -118,19 +118,13 @@ object ExternalShuffleService extends Logging {
server = newShuffleService(sparkConf, securityManager)
server.start()
- installShutdownHook()
+ ShutdownHookManager.addShutdownHook { () =>
+ logInfo("Shutting down shuffle service.")
+ server.stop()
+ barrier.countDown()
+ }
// keep running until the process is terminated
barrier.await()
}
-
- private def installShutdownHook(): Unit = {
- Runtime.getRuntime.addShutdownHook(new Thread("External Shuffle Service shutdown thread") {
- override def run() {
- logInfo("Shutting down shuffle service.")
- server.stop()
- barrier.countDown()
- }
- })
- }
}