This is an automated email from the ASF dual-hosted git repository. dongjoon pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/spark.git
The following commit(s) were added to refs/heads/master by this push: new 81f928a [SPARK-38594][CORE][R] Use `NettyUtils` to create `EventLoopGroup` and `ChannelClass` in `RBackend` 81f928a is described below commit 81f928adf5fcbe089e607c85c267f482388ec1d2 Author: yangjie01 <yangji...@baidu.com> AuthorDate: Sat Apr 2 21:24:17 2022 -0700 [SPARK-38594][CORE][R] Use `NettyUtils` to create `EventLoopGroup` and `ChannelClass` in `RBackend` ### What changes were proposed in this pull request? Spark provides `NettyUtils` to create `EventLoopGroup` and `ChannelClass`, so this pr change to use `NettyUtils` instead of handwriting. ### Why are the changes needed? Unified use of `NettyUtils ### Does this PR introduce _any_ user-facing change? No. ### How was this patch tested? Pass GA Closes #35910 from LuciferYang/rbackend. Authored-by: yangjie01 <yangji...@baidu.com> Signed-off-by: Dongjoon Hyun <dongj...@apache.org> --- core/src/main/scala/org/apache/spark/api/r/RBackend.scala | 8 ++++---- 1 file changed, 4 insertions(+), 4 deletions(-) diff --git a/core/src/main/scala/org/apache/spark/api/r/RBackend.scala b/core/src/main/scala/org/apache/spark/api/r/RBackend.scala index c755dcb..27f80b8 100644 --- a/core/src/main/scala/org/apache/spark/api/r/RBackend.scala +++ b/core/src/main/scala/org/apache/spark/api/r/RBackend.scala @@ -23,9 +23,7 @@ import java.util.concurrent.TimeUnit import io.netty.bootstrap.ServerBootstrap import io.netty.channel.{ChannelFuture, ChannelInitializer, EventLoopGroup} -import io.netty.channel.nio.NioEventLoopGroup import io.netty.channel.socket.SocketChannel -import io.netty.channel.socket.nio.NioServerSocketChannel import io.netty.handler.codec.LengthFieldBasedFrameDecoder import io.netty.handler.codec.bytes.{ByteArrayDecoder, ByteArrayEncoder} import io.netty.handler.timeout.ReadTimeoutHandler @@ -33,6 +31,7 @@ import io.netty.handler.timeout.ReadTimeoutHandler import org.apache.spark.{SparkConf, SparkEnv} import org.apache.spark.internal.Logging import org.apache.spark.internal.config.R._ +import org.apache.spark.network.util.{IOMode, NettyUtils} /** * Netty-based backend server that is used to communicate between R and Java. @@ -49,14 +48,15 @@ private[spark] class RBackend { def init(): (Int, RAuthHelper) = { val conf = Option(SparkEnv.get).map(_.conf).getOrElse(new SparkConf()) val backendConnectionTimeout = conf.get(R_BACKEND_CONNECTION_TIMEOUT) - bossGroup = new NioEventLoopGroup(conf.get(R_NUM_BACKEND_THREADS)) + bossGroup = NettyUtils.createEventLoop(IOMode.NIO, conf.get(R_NUM_BACKEND_THREADS), "RBackend") val workerGroup = bossGroup val handler = new RBackendHandler(this) val authHelper = new RAuthHelper(conf) + val channelClass = NettyUtils.getServerChannelClass(IOMode.NIO) bootstrap = new ServerBootstrap() .group(bossGroup, workerGroup) - .channel(classOf[NioServerSocketChannel]) + .channel(channelClass) bootstrap.childHandler(new ChannelInitializer[SocketChannel]() { def initChannel(ch: SocketChannel): Unit = { --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@spark.apache.org For additional commands, e-mail: commits-h...@spark.apache.org