HBASE-16335 RpcClient under heavy load leaks some netty bytebuf (Ram)

This commit is contained in:
Ramkrishna 2016-09-19 16:12:15 +05:30
parent 6eb6225456
commit c5b8aababe
5 changed files with 53 additions and 15 deletions

View File

@ -215,6 +215,7 @@ public abstract class AbstractRpcClient<T extends RpcConnection> implements RpcC
if (conn.getLastTouched() < closeBeforeTime && !conn.isActive()) { if (conn.getLastTouched() < closeBeforeTime && !conn.isActive()) {
LOG.info("Cleanup idle connection to " + conn.remoteId().address); LOG.info("Cleanup idle connection to " + conn.remoteId().address);
connections.removeValue(conn.remoteId(), conn); connections.removeValue(conn.remoteId(), conn);
conn.cleanupConnection();
} }
} }
} }
@ -472,6 +473,9 @@ public abstract class AbstractRpcClient<T extends RpcConnection> implements RpcC
conn.shutdown(); conn.shutdown();
} }
closeInternal(); closeInternal();
for (T conn : connToClose) {
conn.cleanupConnection();
}
} }
@Override @Override

View File

@ -684,6 +684,11 @@ class BlockingRpcConnection extends RpcConnection implements Runnable {
closeConn(new IOException("connection to " + remoteId.address + " closed")); closeConn(new IOException("connection to " + remoteId.address + " closed"));
} }
@Override
public void cleanupConnection() {
// do nothing
}
@Override @Override
public synchronized void sendRequest(final Call call, HBaseRpcController pcrc) public synchronized void sendRequest(final Call call, HBaseRpcController pcrc)
throws IOException { throws IOException {

View File

@ -36,6 +36,7 @@ import io.netty.channel.ChannelOption;
import io.netty.channel.ChannelPipeline; import io.netty.channel.ChannelPipeline;
import io.netty.handler.codec.LengthFieldBasedFrameDecoder; import io.netty.handler.codec.LengthFieldBasedFrameDecoder;
import io.netty.handler.timeout.IdleStateHandler; import io.netty.handler.timeout.IdleStateHandler;
import io.netty.util.ReferenceCountUtil;
import io.netty.util.concurrent.Future; import io.netty.util.concurrent.Future;
import io.netty.util.concurrent.FutureListener; import io.netty.util.concurrent.FutureListener;
import io.netty.util.concurrent.Promise; import io.netty.util.concurrent.Promise;
@ -119,6 +120,16 @@ class NettyRpcConnection extends RpcConnection {
shutdown0(); shutdown0();
} }
@Override
public synchronized void cleanupConnection() {
if (connectionHeaderPreamble != null) {
ReferenceCountUtil.safeRelease(connectionHeaderPreamble);
}
if (connectionHeaderWithLength != null) {
ReferenceCountUtil.safeRelease(connectionHeaderWithLength);
}
}
private void established(Channel ch) { private void established(Channel ch) {
ch.write(connectionHeaderWithLength.retainedDuplicate()); ch.write(connectionHeaderWithLength.retainedDuplicate());
ChannelPipeline p = ch.pipeline(); ChannelPipeline p = ch.pipeline();

View File

@ -252,4 +252,9 @@ abstract class RpcConnection {
public abstract void shutdown(); public abstract void shutdown();
public abstract void sendRequest(Call call, HBaseRpcController hrc) throws IOException; public abstract void sendRequest(Call call, HBaseRpcController hrc) throws IOException;
/**
* Does the clean up work after the connection is removed from the connection pool
*/
public abstract void cleanupConnection();
} }

View File

@ -23,6 +23,7 @@ import io.netty.channel.ChannelHandlerContext;
import io.netty.channel.ChannelOutboundHandlerAdapter; import io.netty.channel.ChannelOutboundHandlerAdapter;
import io.netty.channel.ChannelPromise; import io.netty.channel.ChannelPromise;
import io.netty.channel.CoalescingBufferQueue; import io.netty.channel.CoalescingBufferQueue;
import io.netty.util.ReferenceCountUtil;
import io.netty.util.concurrent.PromiseCombiner; import io.netty.util.concurrent.PromiseCombiner;
import javax.security.sasl.SaslClient; import javax.security.sasl.SaslClient;
@ -60,10 +61,12 @@ public class SaslWrapHandler extends ChannelOutboundHandlerAdapter {
@Override @Override
public void flush(ChannelHandlerContext ctx) throws Exception { public void flush(ChannelHandlerContext ctx) throws Exception {
ByteBuf buf = null;
try {
if (!queue.isEmpty()) { if (!queue.isEmpty()) {
ChannelPromise promise = ctx.newPromise(); ChannelPromise promise = ctx.newPromise();
int readableBytes = queue.readableBytes(); int readableBytes = queue.readableBytes();
ByteBuf buf = queue.remove(readableBytes, promise); buf = queue.remove(readableBytes, promise);
byte[] bytes = new byte[readableBytes]; byte[] bytes = new byte[readableBytes];
buf.readBytes(bytes); buf.readBytes(bytes);
byte[] wrapperBytes = saslClient.wrap(bytes, 0, bytes.length); byte[] wrapperBytes = saslClient.wrap(bytes, 0, bytes.length);
@ -76,5 +79,15 @@ public class SaslWrapHandler extends ChannelOutboundHandlerAdapter {
combiner.finish(promise); combiner.finish(promise);
} }
ctx.flush(); ctx.flush();
} finally {
if (buf != null) {
ReferenceCountUtil.safeRelease(buf);
}
}
}
@Override
public void close(ChannelHandlerContext ctx, ChannelPromise promise) throws Exception {
queue.releaseAndFailAll(new Throwable("Closed"));
} }
} }