aboutsummaryrefslogtreecommitdiff
path: root/core
diff options
context:
space:
mode:
authorYonathan Randolph <yonathan@liftigniter.com>2016-03-16 09:34:04 +0000
committerSean Owen <sowen@cloudera.com>2016-03-16 09:34:04 +0000
commit05ab2948ab357fc07222bb3505df80b1886f7310 (patch)
tree7a2f7dcaf1293935702e5381c4bdfbb6ec73ee8a /core
parent431a3d04b437df4ab323fd925f57873aa6b8a0c8 (diff)
downloadspark-05ab2948ab357fc07222bb3505df80b1886f7310.tar.gz
spark-05ab2948ab357fc07222bb3505df80b1886f7310.tar.bz2
spark-05ab2948ab357fc07222bb3505df80b1886f7310.zip
[SPARK-13906] Ensure that there are at least 2 dispatcher threads.
## What changes were proposed in this pull request? Force at least two dispatcher-event-loop threads. Since SparkDeploySchedulerBackend (in AppClient) calls askWithRetry to CoarseGrainedScheduler in the same process, there the driver needs at least two dispatcher threads to prevent the dispatcher thread from hanging. ## How was this patch tested? Manual. Author: Yonathan Randolph <yonathangmail.com> Author: Yonathan Randolph <yonathan@liftigniter.com> Closes #11728 from yonran/SPARK-13906.
Diffstat (limited to 'core')
-rw-r--r--core/src/main/scala/org/apache/spark/rpc/netty/Dispatcher.scala2
1 files changed, 1 insertions, 1 deletions
diff --git a/core/src/main/scala/org/apache/spark/rpc/netty/Dispatcher.scala b/core/src/main/scala/org/apache/spark/rpc/netty/Dispatcher.scala
index f092f1d7f6..613d6ee781 100644
--- a/core/src/main/scala/org/apache/spark/rpc/netty/Dispatcher.scala
+++ b/core/src/main/scala/org/apache/spark/rpc/netty/Dispatcher.scala
@@ -192,7 +192,7 @@ private[netty] class Dispatcher(nettyEnv: NettyRpcEnv) extends Logging {
/** Thread pool used for dispatching messages. */
private val threadpool: ThreadPoolExecutor = {
val numThreads = nettyEnv.conf.getInt("spark.rpc.netty.dispatcher.numThreads",
- Runtime.getRuntime.availableProcessors())
+ math.max(2, Runtime.getRuntime.availableProcessors()))
val pool = ThreadUtils.newDaemonFixedThreadPool(numThreads, "dispatcher-event-loop")
for (i <- 0 until numThreads) {
pool.execute(new MessageLoop)