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