Browse Source

[Improvement][remote] improvement netty eventLoopGroup (#3580)

* [Improvement][remote] improvement netty eventLoopGroup
If you are running on linux you can use EpollEventLoopGroup and so get better performance, less GC and have more advanced features that are only available on linux.

* reformat code

* utility classes should not have public constructors

* reformat code

* reformat code

* reformat code

* add test

* code style

* add test

* add test

* fix test error

* add test

* test

* test

* add test config

* add test config
pull/3/MERGE
CalvinKirs 4 years ago committed by GitHub
parent
commit
34a04acafd
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
  1. 78
      dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/NettyRemotingClient.java
  2. 45
      dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/NettyRemotingServer.java
  3. 10
      dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/utils/Constants.java
  4. 42
      dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/utils/NettyUtils.java
  5. 36
      dolphinscheduler-remote/src/test/java/org/apache/dolphinscheduler/remote/NettyUtilTest.java
  6. 3
      pom.xml

78
dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/NettyRemotingClient.java

@ -18,10 +18,17 @@
package org.apache.dolphinscheduler.remote; package org.apache.dolphinscheduler.remote;
import io.netty.bootstrap.Bootstrap; import io.netty.bootstrap.Bootstrap;
import io.netty.channel.*; import io.netty.channel.Channel;
import io.netty.channel.ChannelFuture;
import io.netty.channel.ChannelFutureListener;
import io.netty.channel.ChannelInitializer;
import io.netty.channel.ChannelOption;
import io.netty.channel.EventLoopGroup;
import io.netty.channel.epoll.EpollEventLoopGroup;
import io.netty.channel.nio.NioEventLoopGroup; import io.netty.channel.nio.NioEventLoopGroup;
import io.netty.channel.socket.SocketChannel; import io.netty.channel.socket.SocketChannel;
import io.netty.channel.socket.nio.NioSocketChannel; import io.netty.channel.socket.nio.NioSocketChannel;
import org.apache.dolphinscheduler.remote.codec.NettyDecoder; import org.apache.dolphinscheduler.remote.codec.NettyDecoder;
import org.apache.dolphinscheduler.remote.codec.NettyEncoder; import org.apache.dolphinscheduler.remote.codec.NettyEncoder;
import org.apache.dolphinscheduler.remote.command.Command; import org.apache.dolphinscheduler.remote.command.Command;
@ -38,6 +45,8 @@ import org.apache.dolphinscheduler.remote.processor.NettyRequestProcessor;
import org.apache.dolphinscheduler.remote.utils.Host; import org.apache.dolphinscheduler.remote.utils.Host;
import org.apache.dolphinscheduler.remote.utils.CallerThreadExecutePolicy; import org.apache.dolphinscheduler.remote.utils.CallerThreadExecutePolicy;
import org.apache.dolphinscheduler.remote.utils.NamedThreadFactory; import org.apache.dolphinscheduler.remote.utils.NamedThreadFactory;
import org.apache.dolphinscheduler.remote.utils.NettyUtils;
import org.slf4j.Logger; import org.slf4j.Logger;
import org.slf4j.LoggerFactory; import org.slf4j.LoggerFactory;
@ -76,7 +85,7 @@ public class NettyRemotingClient {
/** /**
* worker group * worker group
*/ */
private final NioEventLoopGroup workerGroup; private final EventLoopGroup workerGroup;
/** /**
* client config * client config
@ -105,10 +114,21 @@ public class NettyRemotingClient {
/** /**
* client init * client init
*
* @param clientConfig client config * @param clientConfig client config
*/ */
public NettyRemotingClient(final NettyClientConfig clientConfig){ public NettyRemotingClient(final NettyClientConfig clientConfig) {
this.clientConfig = clientConfig; this.clientConfig = clientConfig;
if (NettyUtils.useEpoll()) {
this.workerGroup = new EpollEventLoopGroup(clientConfig.getWorkerThreads(), new ThreadFactory() {
private AtomicInteger threadIndex = new AtomicInteger(0);
@Override
public Thread newThread(Runnable r) {
return new Thread(r, String.format("NettyClient_%d", this.threadIndex.incrementAndGet()));
}
});
} else {
this.workerGroup = new NioEventLoopGroup(clientConfig.getWorkerThreads(), new ThreadFactory() { this.workerGroup = new NioEventLoopGroup(clientConfig.getWorkerThreads(), new ThreadFactory() {
private AtomicInteger threadIndex = new AtomicInteger(0); private AtomicInteger threadIndex = new AtomicInteger(0);
@ -117,6 +137,7 @@ public class NettyRemotingClient {
return new Thread(r, String.format("NettyClient_%d", this.threadIndex.incrementAndGet())); return new Thread(r, String.format("NettyClient_%d", this.threadIndex.incrementAndGet()));
} }
}); });
}
this.callbackExecutor = new ThreadPoolExecutor(5, 10, 1, TimeUnit.MINUTES, this.callbackExecutor = new ThreadPoolExecutor(5, 10, 1, TimeUnit.MINUTES,
new LinkedBlockingQueue<>(1000), new NamedThreadFactory("CallbackExecutor", 10), new LinkedBlockingQueue<>(1000), new NamedThreadFactory("CallbackExecutor", 10),
new CallerThreadExecutePolicy()); new CallerThreadExecutePolicy());
@ -130,7 +151,7 @@ public class NettyRemotingClient {
/** /**
* start * start
*/ */
private void start(){ private void start() {
this.bootstrap this.bootstrap
.group(this.workerGroup) .group(this.workerGroup)
@ -160,6 +181,7 @@ public class NettyRemotingClient {
/** /**
* async send * async send
*
* @param host host * @param host host
* @param command command * @param command command
* @param timeoutMillis timeoutMillis * @param timeoutMillis timeoutMillis
@ -182,7 +204,7 @@ public class NettyRemotingClient {
* control concurrency number * control concurrency number
*/ */
boolean acquired = this.asyncSemaphore.tryAcquire(timeoutMillis, TimeUnit.MILLISECONDS); boolean acquired = this.asyncSemaphore.tryAcquire(timeoutMillis, TimeUnit.MILLISECONDS);
if(acquired){ if (acquired) {
final ReleaseSemaphore releaseSemaphore = new ReleaseSemaphore(this.asyncSemaphore); final ReleaseSemaphore releaseSemaphore = new ReleaseSemaphore(this.asyncSemaphore);
/** /**
@ -193,11 +215,11 @@ public class NettyRemotingClient {
invokeCallback, invokeCallback,
releaseSemaphore); releaseSemaphore);
try { try {
channel.writeAndFlush(command).addListener(new ChannelFutureListener(){ channel.writeAndFlush(command).addListener(new ChannelFutureListener() {
@Override @Override
public void operationComplete(ChannelFuture future) throws Exception { public void operationComplete(ChannelFuture future) throws Exception {
if(future.isSuccess()){ if (future.isSuccess()) {
responseFuture.setSendOk(true); responseFuture.setSendOk(true);
return; return;
} else { } else {
@ -207,18 +229,18 @@ public class NettyRemotingClient {
responseFuture.putResponse(null); responseFuture.putResponse(null);
try { try {
responseFuture.executeInvokeCallback(); responseFuture.executeInvokeCallback();
} catch (Throwable ex){ } catch (Throwable ex) {
logger.error("execute callback error", ex); logger.error("execute callback error", ex);
} finally{ } finally {
responseFuture.release(); responseFuture.release();
} }
} }
}); });
} catch (Throwable ex){ } catch (Throwable ex) {
responseFuture.release(); responseFuture.release();
throw new RemotingException(String.format("send command to host: %s failed", host), ex); throw new RemotingException(String.format("send command to host: %s failed", host), ex);
} }
} else{ } else {
String message = String.format("try to acquire async semaphore timeout: %d, waiting thread num: %d, total permits: %d", String message = String.format("try to acquire async semaphore timeout: %d, waiting thread num: %d, total permits: %d",
timeoutMillis, asyncSemaphore.getQueueLength(), asyncSemaphore.availablePermits()); timeoutMillis, asyncSemaphore.getQueueLength(), asyncSemaphore.availablePermits());
throw new RemotingTooMuchRequestException(message); throw new RemotingTooMuchRequestException(message);
@ -227,6 +249,7 @@ public class NettyRemotingClient {
/** /**
* sync send * sync send
*
* @param host host * @param host host
* @param command command * @param command command
* @param timeoutMillis timeoutMillis * @param timeoutMillis timeoutMillis
@ -244,7 +267,7 @@ public class NettyRemotingClient {
channel.writeAndFlush(command).addListener(new ChannelFutureListener() { channel.writeAndFlush(command).addListener(new ChannelFutureListener() {
@Override @Override
public void operationComplete(ChannelFuture future) throws Exception { public void operationComplete(ChannelFuture future) throws Exception {
if(future.isSuccess()){ if (future.isSuccess()) {
responseFuture.setSendOk(true); responseFuture.setSendOk(true);
return; return;
} else { } else {
@ -259,10 +282,10 @@ public class NettyRemotingClient {
* sync wait for result * sync wait for result
*/ */
Command result = responseFuture.waitResponse(); Command result = responseFuture.waitResponse();
if(result == null){ if (result == null) {
if(responseFuture.isSendOK()){ if (responseFuture.isSendOK()) {
throw new RemotingTimeoutException(host.toString(), timeoutMillis, responseFuture.getCause()); throw new RemotingTimeoutException(host.toString(), timeoutMillis, responseFuture.getCause());
} else{ } else {
throw new RemotingException(host.toString(), responseFuture.getCause()); throw new RemotingException(host.toString(), responseFuture.getCause());
} }
} }
@ -271,6 +294,7 @@ public class NettyRemotingClient {
/** /**
* send task * send task
*
* @param host host * @param host host
* @param command command * @param command command
* @throws RemotingException * @throws RemotingException
@ -297,6 +321,7 @@ public class NettyRemotingClient {
/** /**
* register processor * register processor
*
* @param commandType command type * @param commandType command type
* @param processor processor * @param processor processor
*/ */
@ -317,12 +342,13 @@ public class NettyRemotingClient {
/** /**
* get channel * get channel
*
* @param host * @param host
* @return * @return
*/ */
public Channel getChannel(Host host) { public Channel getChannel(Host host) {
Channel channel = channels.get(host); Channel channel = channels.get(host);
if(channel != null && channel.isActive()){ if (channel != null && channel.isActive()) {
return channel; return channel;
} }
return createChannel(host, true); return createChannel(host, true);
@ -330,6 +356,7 @@ public class NettyRemotingClient {
/** /**
* create channel * create channel
*
* @param host host * @param host host
* @param isSync sync flag * @param isSync sync flag
* @return channel * @return channel
@ -337,10 +364,10 @@ public class NettyRemotingClient {
public Channel createChannel(Host host, boolean isSync) { public Channel createChannel(Host host, boolean isSync) {
ChannelFuture future; ChannelFuture future;
try { try {
synchronized (bootstrap){ synchronized (bootstrap) {
future = bootstrap.connect(new InetSocketAddress(host.getIp(), host.getPort())); future = bootstrap.connect(new InetSocketAddress(host.getIp(), host.getPort()));
} }
if(isSync){ if (isSync) {
future.sync(); future.sync();
} }
if (future.isSuccess()) { if (future.isSuccess()) {
@ -358,16 +385,16 @@ public class NettyRemotingClient {
* close * close
*/ */
public void close() { public void close() {
if(isStarted.compareAndSet(true, false)){ if (isStarted.compareAndSet(true, false)) {
try { try {
closeChannels(); closeChannels();
if(workerGroup != null){ if (workerGroup != null) {
this.workerGroup.shutdownGracefully(); this.workerGroup.shutdownGracefully();
} }
if(callbackExecutor != null){ if (callbackExecutor != null) {
this.callbackExecutor.shutdownNow(); this.callbackExecutor.shutdownNow();
} }
if(this.responseFutureExecutor != null){ if (this.responseFutureExecutor != null) {
this.responseFutureExecutor.shutdownNow(); this.responseFutureExecutor.shutdownNow();
} }
} catch (Exception ex) { } catch (Exception ex) {
@ -380,7 +407,7 @@ public class NettyRemotingClient {
/** /**
* close channels * close channels
*/ */
private void closeChannels(){ private void closeChannels() {
for (Channel channel : this.channels.values()) { for (Channel channel : this.channels.values()) {
channel.close(); channel.close();
} }
@ -389,11 +416,12 @@ public class NettyRemotingClient {
/** /**
* close channel * close channel
*
* @param host host * @param host host
*/ */
public void closeChannel(Host host){ public void closeChannel(Host host) {
Channel channel = this.channels.remove(host); Channel channel = this.channels.remove(host);
if(channel != null){ if (channel != null) {
channel.close(); channel.close();
} }
} }

45
dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/NettyRemotingServer.java

@ -22,9 +22,12 @@ import io.netty.channel.ChannelFuture;
import io.netty.channel.ChannelInitializer; import io.netty.channel.ChannelInitializer;
import io.netty.channel.ChannelOption; import io.netty.channel.ChannelOption;
import io.netty.channel.ChannelPipeline; import io.netty.channel.ChannelPipeline;
import io.netty.channel.EventLoopGroup;
import io.netty.channel.epoll.EpollEventLoopGroup;
import io.netty.channel.nio.NioEventLoopGroup; import io.netty.channel.nio.NioEventLoopGroup;
import io.netty.channel.socket.nio.NioServerSocketChannel; import io.netty.channel.socket.nio.NioServerSocketChannel;
import io.netty.channel.socket.nio.NioSocketChannel; import io.netty.channel.socket.nio.NioSocketChannel;
import org.apache.dolphinscheduler.remote.codec.NettyDecoder; import org.apache.dolphinscheduler.remote.codec.NettyDecoder;
import org.apache.dolphinscheduler.remote.codec.NettyEncoder; import org.apache.dolphinscheduler.remote.codec.NettyEncoder;
import org.apache.dolphinscheduler.remote.command.CommandType; import org.apache.dolphinscheduler.remote.command.CommandType;
@ -32,6 +35,8 @@ import org.apache.dolphinscheduler.remote.config.NettyServerConfig;
import org.apache.dolphinscheduler.remote.handler.NettyServerHandler; import org.apache.dolphinscheduler.remote.handler.NettyServerHandler;
import org.apache.dolphinscheduler.remote.processor.NettyRequestProcessor; import org.apache.dolphinscheduler.remote.processor.NettyRequestProcessor;
import org.apache.dolphinscheduler.remote.utils.Constants; import org.apache.dolphinscheduler.remote.utils.Constants;
import org.apache.dolphinscheduler.remote.utils.NettyUtils;
import org.slf4j.Logger; import org.slf4j.Logger;
import org.slf4j.LoggerFactory; import org.slf4j.LoggerFactory;
@ -66,12 +71,12 @@ public class NettyRemotingServer {
/** /**
* boss group * boss group
*/ */
private final NioEventLoopGroup bossGroup; private final EventLoopGroup bossGroup;
/** /**
* worker group * worker group
*/ */
private final NioEventLoopGroup workGroup; private final EventLoopGroup workGroup;
/** /**
* server config * server config
@ -93,9 +98,27 @@ public class NettyRemotingServer {
* *
* @param serverConfig server config * @param serverConfig server config
*/ */
public NettyRemotingServer(final NettyServerConfig serverConfig){ public NettyRemotingServer(final NettyServerConfig serverConfig) {
this.serverConfig = serverConfig; this.serverConfig = serverConfig;
if (NettyUtils.useEpoll()) {
this.bossGroup = new EpollEventLoopGroup(1, new ThreadFactory() {
private AtomicInteger threadIndex = new AtomicInteger(0);
@Override
public Thread newThread(Runnable r) {
return new Thread(r, String.format("NettyServerBossThread_%d", this.threadIndex.incrementAndGet()));
}
});
this.workGroup = new EpollEventLoopGroup(serverConfig.getWorkerThread(), new ThreadFactory() {
private AtomicInteger threadIndex = new AtomicInteger(0);
@Override
public Thread newThread(Runnable r) {
return new Thread(r, String.format("NettyServerWorkerThread_%d", this.threadIndex.incrementAndGet()));
}
});
} else {
this.bossGroup = new NioEventLoopGroup(1, new ThreadFactory() { this.bossGroup = new NioEventLoopGroup(1, new ThreadFactory() {
private AtomicInteger threadIndex = new AtomicInteger(0); private AtomicInteger threadIndex = new AtomicInteger(0);
@ -114,11 +137,12 @@ public class NettyRemotingServer {
} }
}); });
} }
}
/** /**
* server start * server start
*/ */
public void start(){ public void start() {
if (isStarted.compareAndSet(false, true)) { if (isStarted.compareAndSet(false, true)) {
this.serverBootstrap this.serverBootstrap
.group(this.bossGroup, this.workGroup) .group(this.bossGroup, this.workGroup)
@ -141,7 +165,7 @@ public class NettyRemotingServer {
try { try {
future = serverBootstrap.bind(serverConfig.getListenPort()).sync(); future = serverBootstrap.bind(serverConfig.getListenPort()).sync();
} catch (Exception e) { } catch (Exception e) {
logger.error("NettyRemotingServer bind fail {}, exit",e.getMessage(), e); logger.error("NettyRemotingServer bind fail {}, exit", e.getMessage(), e);
throw new RuntimeException(String.format("NettyRemotingServer bind %s fail", serverConfig.getListenPort())); throw new RuntimeException(String.format("NettyRemotingServer bind %s fail", serverConfig.getListenPort()));
} }
if (future.isSuccess()) { if (future.isSuccess()) {
@ -156,10 +180,11 @@ public class NettyRemotingServer {
/** /**
* init netty channel * init netty channel
*
* @param ch socket channel * @param ch socket channel
* @throws Exception * @throws Exception
*/ */
private void initNettyChannel(NioSocketChannel ch) throws Exception{ private void initNettyChannel(NioSocketChannel ch) throws Exception {
ChannelPipeline pipeline = ch.pipeline(); ChannelPipeline pipeline = ch.pipeline();
pipeline.addLast("encoder", encoder); pipeline.addLast("encoder", encoder);
pipeline.addLast("decoder", new NettyDecoder()); pipeline.addLast("decoder", new NettyDecoder());
@ -168,6 +193,7 @@ public class NettyRemotingServer {
/** /**
* register processor * register processor
*
* @param commandType command type * @param commandType command type
* @param processor processor * @param processor processor
*/ */
@ -188,6 +214,7 @@ public class NettyRemotingServer {
/** /**
* get default thread executor * get default thread executor
*
* @return thread executor * @return thread executor
*/ */
public ExecutorService getDefaultExecutor() { public ExecutorService getDefaultExecutor() {
@ -195,12 +222,12 @@ public class NettyRemotingServer {
} }
public void close() { public void close() {
if(isStarted.compareAndSet(true, false)){ if (isStarted.compareAndSet(true, false)) {
try { try {
if(bossGroup != null){ if (bossGroup != null) {
this.bossGroup.shutdownGracefully(); this.bossGroup.shutdownGracefully();
} }
if(workGroup != null){ if (workGroup != null) {
this.workGroup.shutdownGracefully(); this.workGroup.shutdownGracefully();
} }
defaultExecutor.shutdown(); defaultExecutor.shutdown();

10
dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/utils/Constants.java

@ -42,4 +42,14 @@ public class Constants {
public static final String LOCAL_ADDRESS = IPUtils.getFirstNoLoopbackIP4Address(); public static final String LOCAL_ADDRESS = IPUtils.getFirstNoLoopbackIP4Address();
/**
* netty epoll enable switch
*/
public static final String NETTY_EPOLL_ENABLE = System.getProperty("netty.epoll.enable");
/**
* OS Name
*/
public static final String OS_NAME = System.getProperty("os.name");
} }

42
dolphinscheduler-remote/src/main/java/org/apache/dolphinscheduler/remote/utils/NettyUtils.java

@ -0,0 +1,42 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.dolphinscheduler.remote.utils;
import io.netty.channel.epoll.Epoll;
/**
* NettyUtils
*/
public class NettyUtils {
private NettyUtils() {
}
public static boolean useEpoll() {
String osName = Constants.OS_NAME;
if (!osName.toLowerCase().contains("linux")) {
return false;
}
if (!Epoll.isAvailable()) {
return false;
}
String enableNettyEpoll = Constants.NETTY_EPOLL_ENABLE;
return Boolean.parseBoolean(enableNettyEpoll);
}
}

36
dolphinscheduler-remote/src/test/java/org/apache/dolphinscheduler/remote/NettyUtilTest.java

@ -0,0 +1,36 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.dolphinscheduler.remote;
import org.apache.dolphinscheduler.remote.utils.NettyUtils;
import org.junit.Assert;
import org.junit.Test;
/**
* NettyUtilTest
*/
public class NettyUtilTest {
@Test
public void testUserEpoll() {
System.setProperty("netty.epoll.enable", "false");
Assert.assertFalse(NettyUtils.useEpoll());
}
}

3
pom.xml

@ -804,7 +804,8 @@
<include>**/remote/JsonSerializerTest.java</include> <include>**/remote/JsonSerializerTest.java</include>
<include>**/remote/RemoveTaskLogResponseCommandTest.java</include> <include>**/remote/RemoveTaskLogResponseCommandTest.java</include>
<include>**/remote/RemoveTaskLogRequestCommandTest.java</include> <include>**/remote/RemoveTaskLogRequestCommandTest.java</include>
<!--<include>**/remote/NettyRemotingClientTest.java</include>--> <include>**/remote/NettyRemotingClientTest.java</include>
<include>**/remote/NettyUtilTest.java</include>
<include>**/remote/ResponseFutureTest.java</include> <include>**/remote/ResponseFutureTest.java</include>
<include>**/server/log/LoggerServerTest.java</include> <include>**/server/log/LoggerServerTest.java</include>
<include>**/server/entity/SQLTaskExecutionContextTest.java</include> <include>**/server/entity/SQLTaskExecutionContextTest.java</include>

Loading…
Cancel
Save