srowen closed pull request #23278: [SPARK-24920][Core] Allow sharing Netty's
memory pool allocators
URL: https://github.com/apache/spark/pull/23278
This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:
As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):
diff --git
a/common/network-common/src/main/java/org/apache/spark/network/client/TransportClientFactory.java
b/common/network-common/src/main/java/org/apache/spark/network/client/TransportClientFactory.java
index 16d242dbb2c47..a8e27157f42fb 100644
---
a/common/network-common/src/main/java/org/apache/spark/network/client/TransportClientFactory.java
+++
b/common/network-common/src/main/java/org/apache/spark/network/client/TransportClientFactory.java
@@ -84,7 +84,7 @@
private final Class<? extends Channel> socketChannelClass;
private EventLoopGroup workerGroup;
- private PooledByteBufAllocator pooledAllocator;
+ private final PooledByteBufAllocator pooledAllocator;
private final NettyMemoryMetrics metrics;
public TransportClientFactory(
@@ -103,8 +103,13 @@ public TransportClientFactory(
ioMode,
conf.clientThreads(),
conf.getModuleName() + "-client");
- this.pooledAllocator = NettyUtils.createPooledByteBufAllocator(
- conf.preferDirectBufs(), false /* allowCache */, conf.clientThreads());
+ if (conf.sharedByteBufAllocators()) {
+ this.pooledAllocator = NettyUtils.getSharedPooledByteBufAllocator(
+ conf.preferDirectBufsForSharedByteBufAllocators(), false /*
allowCache */);
+ } else {
+ this.pooledAllocator = NettyUtils.createPooledByteBufAllocator(
+ conf.preferDirectBufs(), false /* allowCache */,
conf.clientThreads());
+ }
this.metrics = new NettyMemoryMetrics(
this.pooledAllocator, conf.getModuleName() + "-client", conf);
}
diff --git
a/common/network-common/src/main/java/org/apache/spark/network/server/TransportServer.java
b/common/network-common/src/main/java/org/apache/spark/network/server/TransportServer.java
index 9c85ab2f5f06f..562439b5fdc04 100644
---
a/common/network-common/src/main/java/org/apache/spark/network/server/TransportServer.java
+++
b/common/network-common/src/main/java/org/apache/spark/network/server/TransportServer.java
@@ -53,6 +53,7 @@
private ServerBootstrap bootstrap;
private ChannelFuture channelFuture;
private int port = -1;
+ private final PooledByteBufAllocator pooledAllocator;
private NettyMemoryMetrics metrics;
/**
@@ -68,6 +69,13 @@ public TransportServer(
this.context = context;
this.conf = context.getConf();
this.appRpcHandler = appRpcHandler;
+ if (conf.sharedByteBufAllocators()) {
+ this.pooledAllocator = NettyUtils.getSharedPooledByteBufAllocator(
+ conf.preferDirectBufsForSharedByteBufAllocators(), true /*
allowCache */);
+ } else {
+ this.pooledAllocator = NettyUtils.createPooledByteBufAllocator(
+ conf.preferDirectBufs(), true /* allowCache */,
conf.serverThreads());
+ }
this.bootstraps =
Lists.newArrayList(Preconditions.checkNotNull(bootstraps));
boolean shouldClose = true;
@@ -95,18 +103,15 @@ private void init(String hostToBind, int portToBind) {
NettyUtils.createEventLoop(ioMode, conf.serverThreads(),
conf.getModuleName() + "-server");
EventLoopGroup workerGroup = bossGroup;
- PooledByteBufAllocator allocator = NettyUtils.createPooledByteBufAllocator(
- conf.preferDirectBufs(), true /* allowCache */, conf.serverThreads());
-
bootstrap = new ServerBootstrap()
.group(bossGroup, workerGroup)
.channel(NettyUtils.getServerChannelClass(ioMode))
- .option(ChannelOption.ALLOCATOR, allocator)
+ .option(ChannelOption.ALLOCATOR, pooledAllocator)
.option(ChannelOption.SO_REUSEADDR, !SystemUtils.IS_OS_WINDOWS)
- .childOption(ChannelOption.ALLOCATOR, allocator);
+ .childOption(ChannelOption.ALLOCATOR, pooledAllocator);
this.metrics = new NettyMemoryMetrics(
- allocator, conf.getModuleName() + "-server", conf);
+ pooledAllocator, conf.getModuleName() + "-server", conf);
if (conf.backLog() > 0) {
bootstrap.option(ChannelOption.SO_BACKLOG, conf.backLog());
diff --git
a/common/network-common/src/main/java/org/apache/spark/network/util/NettyUtils.java
b/common/network-common/src/main/java/org/apache/spark/network/util/NettyUtils.java
index 33d6eb4a83a0c..423cc0c70ea02 100644
---
a/common/network-common/src/main/java/org/apache/spark/network/util/NettyUtils.java
+++
b/common/network-common/src/main/java/org/apache/spark/network/util/NettyUtils.java
@@ -36,6 +36,22 @@
* Utilities for creating various Netty constructs based on whether we're
using EPOLL or NIO.
*/
public class NettyUtils {
+
+ /**
+ * Specifies an upper bound on the number of Netty threads that Spark
requires by default.
+ * In practice, only 2-4 cores should be required to transfer roughly 10
Gb/s, and each core
+ * that we use will have an initial overhead of roughly 32 MB of off-heap
memory, which comes
+ * at a premium.
+ *
+ * Thus, this value should still retain maximum throughput and reduce wasted
off-heap memory
+ * allocation. It can be overridden by setting the number of serverThreads
and clientThreads
+ * manually in Spark's configuration.
+ */
+ private static int MAX_DEFAULT_NETTY_THREADS = 8;
+
+ private static final PooledByteBufAllocator[] _sharedPooledByteBufAllocator =
+ new PooledByteBufAllocator[2];
+
/** Creates a new ThreadFactory which prefixes each thread with the given
name. */
public static ThreadFactory createThreadFactory(String threadPoolPrefix) {
return new DefaultThreadFactory(threadPoolPrefix, true);
@@ -95,6 +111,38 @@ public static String getRemoteAddress(Channel channel) {
return "<unknown remote>";
}
+ /**
+ * Returns the default number of threads for both the Netty client and
server thread pools.
+ * If numUsableCores is 0, we will use Runtime get an approximate number of
available cores.
+ */
+ public static int defaultNumThreads(int numUsableCores) {
+ final int availableCores;
+ if (numUsableCores > 0) {
+ availableCores = numUsableCores;
+ } else {
+ availableCores = Runtime.getRuntime().availableProcessors();
+ }
+ return Math.min(availableCores, MAX_DEFAULT_NETTY_THREADS);
+ }
+
+ /**
+ * Returns the lazily created shared pooled ByteBuf allocator for the
specified allowCache
+ * parameter value.
+ */
+ public static synchronized PooledByteBufAllocator
getSharedPooledByteBufAllocator(
+ boolean allowDirectBufs,
+ boolean allowCache) {
+ final int index = allowCache ? 0 : 1;
+ if (_sharedPooledByteBufAllocator[index] == null) {
+ _sharedPooledByteBufAllocator[index] =
+ createPooledByteBufAllocator(
+ allowDirectBufs,
+ allowCache,
+ defaultNumThreads(0));
+ }
+ return _sharedPooledByteBufAllocator[index];
+ }
+
/**
* Create a pooled ByteBuf allocator but disables the thread-local cache.
Thread-local caches
* are disabled for TransportClients because the ByteBufs are allocated by
the event loop thread,
diff --git
a/common/network-common/src/main/java/org/apache/spark/network/util/TransportConf.java
b/common/network-common/src/main/java/org/apache/spark/network/util/TransportConf.java
index 43a6bc7dc3d06..73e163b03f0a7 100644
---
a/common/network-common/src/main/java/org/apache/spark/network/util/TransportConf.java
+++
b/common/network-common/src/main/java/org/apache/spark/network/util/TransportConf.java
@@ -265,6 +265,23 @@ public boolean saslServerAlwaysEncrypt() {
return conf.getBoolean("spark.network.sasl.serverAlwaysEncrypt", false);
}
+ /**
+ * Flag indicating whether to share the pooled ByteBuf allocators between
the different Netty
+ * channels. If enabled then only two pooled ByteBuf allocators are created:
one where caching
+ * is allowed (for transport servers) and one where not (for transport
clients).
+ * When disabled a new allocator is created for each transport servers and
clients.
+ */
+ public boolean sharedByteBufAllocators() {
+ return conf.getBoolean("spark.network.sharedByteBufAllocators.enabled",
true);
+ }
+
+ /**
+ * If enabled then off-heap byte buffers will be prefered for the shared
ByteBuf allocators.
+ */
+ public boolean preferDirectBufsForSharedByteBufAllocators() {
+ return conf.getBoolean("spark.network.io.preferDirectBufs", true);
+ }
+
/**
* The commons-crypto configuration for the module.
*/
@@ -313,4 +330,5 @@ public int chunkFetchHandlerThreads() {
(this.serverThreads() > 0 ? this.serverThreads() : 2 *
NettyRuntime.availableProcessors()) *
chunkFetchHandlerThreadsPercent/(double)100);
}
+
}
diff --git
a/core/src/main/scala/org/apache/spark/network/netty/SparkTransportConf.scala
b/core/src/main/scala/org/apache/spark/network/netty/SparkTransportConf.scala
index 25f7bcb9801b9..3ba0a0a750f97 100644
---
a/core/src/main/scala/org/apache/spark/network/netty/SparkTransportConf.scala
+++
b/core/src/main/scala/org/apache/spark/network/netty/SparkTransportConf.scala
@@ -20,7 +20,7 @@ package org.apache.spark.network.netty
import scala.collection.JavaConverters._
import org.apache.spark.SparkConf
-import org.apache.spark.network.util.{ConfigProvider, TransportConf}
+import org.apache.spark.network.util.{ConfigProvider, NettyUtils,
TransportConf}
/**
* Provides a utility for transforming from a SparkConf inside a Spark JVM
(e.g., Executor,
@@ -28,17 +28,6 @@ import org.apache.spark.network.util.{ConfigProvider,
TransportConf}
* like the number of cores that are allocated to this JVM.
*/
object SparkTransportConf {
- /**
- * Specifies an upper bound on the number of Netty threads that Spark
requires by default.
- * In practice, only 2-4 cores should be required to transfer roughly 10
Gb/s, and each core
- * that we use will have an initial overhead of roughly 32 MB of off-heap
memory, which comes
- * at a premium.
- *
- * Thus, this value should still retain maximum throughput and reduce wasted
off-heap memory
- * allocation. It can be overridden by setting the number of serverThreads
and clientThreads
- * manually in Spark's configuration.
- */
- private val MAX_DEFAULT_NETTY_THREADS = 8
/**
* Utility for creating a [[TransportConf]] from a [[SparkConf]].
@@ -54,7 +43,7 @@ object SparkTransportConf {
// Specify thread configuration based on our JVM's allocation of cores
(rather than necessarily
// assuming we have all the machine's cores).
// NB: Only set if serverThreads/clientThreads not already set.
- val numThreads = defaultNumThreads(numUsableCores)
+ val numThreads = NettyUtils.defaultNumThreads(numUsableCores)
conf.setIfMissing(s"spark.$module.io.serverThreads", numThreads.toString)
conf.setIfMissing(s"spark.$module.io.clientThreads", numThreads.toString)
@@ -66,14 +55,4 @@ object SparkTransportConf {
}
})
}
-
- /**
- * Returns the default number of threads for both the Netty client and
server thread pools.
- * If numUsableCores is 0, we will use Runtime get an approximate number of
available cores.
- */
- private def defaultNumThreads(numUsableCores: Int): Int = {
- val availableCores =
- if (numUsableCores > 0) numUsableCores else
Runtime.getRuntime.availableProcessors()
- math.min(availableCores, MAX_DEFAULT_NETTY_THREADS)
- }
}
diff --git a/docs/configuration.md b/docs/configuration.md
index 9abbb3f634900..2c229c4f24fbe 100644
--- a/docs/configuration.md
+++ b/docs/configuration.md
@@ -1505,6 +1505,16 @@ Apart from these, the following properties are also
available, and may be useful
<code>spark.rpc.lookupTimeout</code> if they are not configured.
</td>
</tr>
+<tr>
+ <td><code>spark.network.io.preferDirectBufs</code></td>
+ <td>true</td>
+ <td>
+ If enabled then off-heap buffer allocations are preferred by the shared
allocators.
+ Off-heap buffers are used to reduce garbage collection during shuffle and
cache
+ block transfer. For environments where off-heap memory is tightly limited,
users may wish to
+ turn this off to force all allocations to be on-heap.
+ </td>
+</tr>
<tr>
<td><code>spark.port.maxRetries</code></td>
<td>16</td>
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]
With regards,
Apache Git Services
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]