-
Notifications
You must be signed in to change notification settings - Fork 25.6k
Chunk + Throttle Netty Writes #39286
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Changes from all commits
4dbe81b
569492d
b9fd81d
375011e
fbe339c
760bf28
21ef9d1
1e70435
9e9f201
24ec12d
0e8a0ad
2cf079f
7119379
844d9b9
cd9ab1c
6854d74
9234382
30cbf46
de7a7be
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -22,12 +22,17 @@ | |
| import io.netty.buffer.ByteBuf; | ||
| import io.netty.channel.Channel; | ||
| import io.netty.channel.ChannelDuplexHandler; | ||
| import io.netty.channel.ChannelFuture; | ||
| import io.netty.channel.ChannelHandlerContext; | ||
| import io.netty.channel.ChannelPromise; | ||
| import io.netty.util.Attribute; | ||
| import org.elasticsearch.ElasticsearchException; | ||
| import org.elasticsearch.ExceptionsHelper; | ||
| import org.elasticsearch.transport.Transports; | ||
|
|
||
| import java.nio.channels.ClosedChannelException; | ||
| import java.util.ArrayDeque; | ||
| import java.util.Queue; | ||
|
|
||
| /** | ||
| * A handler (must be the last one!) that does size based frame decoding and forwards the actual message | ||
|
|
@@ -37,13 +42,17 @@ final class Netty4MessageChannelHandler extends ChannelDuplexHandler { | |
|
|
||
| private final Netty4Transport transport; | ||
|
|
||
| private final Queue<WriteOperation> queuedWrites = new ArrayDeque<>(); | ||
|
|
||
| private WriteOperation currentWrite; | ||
|
|
||
| Netty4MessageChannelHandler(Netty4Transport transport) { | ||
| this.transport = transport; | ||
| } | ||
|
|
||
| @Override | ||
| public void channelRead(ChannelHandlerContext ctx, Object msg) throws Exception { | ||
| Transports.assertTransportThread(); | ||
| public void channelRead(ChannelHandlerContext ctx, Object msg) { | ||
| assert Transports.assertTransportThread(); | ||
| assert msg instanceof ByteBuf : "Expected message type ByteBuf, found: " + msg.getClass(); | ||
|
|
||
| final ByteBuf buffer = (ByteBuf) msg; | ||
|
|
@@ -57,7 +66,7 @@ public void channelRead(ChannelHandlerContext ctx, Object msg) throws Exception | |
| } | ||
|
|
||
| @Override | ||
| public void exceptionCaught(ChannelHandlerContext ctx, Throwable cause) throws Exception { | ||
| public void exceptionCaught(ChannelHandlerContext ctx, Throwable cause) { | ||
| ExceptionsHelper.maybeDieOnAnotherThread(cause); | ||
| final Throwable unwrapped = ExceptionsHelper.unwrap(cause, ElasticsearchException.class); | ||
| final Throwable newCause = unwrapped != null ? unwrapped : cause; | ||
|
|
@@ -68,4 +77,113 @@ public void exceptionCaught(ChannelHandlerContext ctx, Throwable cause) throws E | |
| transport.onException(tcpChannel, (Exception) newCause); | ||
| } | ||
| } | ||
|
|
||
| @Override | ||
| public void write(ChannelHandlerContext ctx, Object msg, ChannelPromise promise) { | ||
| assert msg instanceof ByteBuf; | ||
| final boolean queued = queuedWrites.offer(new WriteOperation((ByteBuf) msg, promise)); | ||
| assert queued; | ||
| } | ||
|
|
||
| @Override | ||
| public void channelWritabilityChanged(ChannelHandlerContext ctx) { | ||
| if (ctx.channel().isWritable()) { | ||
| doFlush(ctx); | ||
| } | ||
| ctx.fireChannelWritabilityChanged(); | ||
| } | ||
|
|
||
| @Override | ||
| public void flush(ChannelHandlerContext ctx) { | ||
| Channel channel = ctx.channel(); | ||
| if (channel.isWritable() || channel.isActive() == false) { | ||
| doFlush(ctx); | ||
| } | ||
| } | ||
|
|
||
| @Override | ||
| public void channelInactive(ChannelHandlerContext ctx) throws Exception { | ||
|
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I find this a bit suspicious since channelInactive is from ChannelInboundHandler?
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more.
|
||
| doFlush(ctx); | ||
| super.channelInactive(ctx); | ||
| } | ||
|
|
||
| private void doFlush(ChannelHandlerContext ctx) { | ||
| assert ctx.executor().inEventLoop(); | ||
| final Channel channel = ctx.channel(); | ||
| if (channel.isActive() == false) { | ||
| if (currentWrite != null) { | ||
| currentWrite.promise.tryFailure(new ClosedChannelException()); | ||
| } | ||
|
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Maybe set currentWrite to null to only trigger once?
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. That's ok |
||
| failQueuedWrites(); | ||
| return; | ||
| } | ||
| while (channel.isWritable()) { | ||
| if (currentWrite == null) { | ||
|
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Are we sure that there is only one thread in here at a time? If not, we need to guard the reads/writes to currentWrite.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. We shouldn't be calling |
||
| currentWrite = queuedWrites.poll(); | ||
| } | ||
| if (currentWrite == null) { | ||
| break; | ||
| } | ||
| final WriteOperation write = currentWrite; | ||
|
Contributor
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I think you will need to guard against the promise from already failing. The prior channel write could have failed and failed the the promise associated with the write operation. I imagine that this will kill the channel and cause everything to shutdown. But it still seems safest to prevent potentially flushing bytes that have already been released.
Contributor
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. That's why I put the cleanup loop that you mentioned above. This way we def. fail all the writes and drain the queue on failures before going for another write in the loop. And since the only way, the callbacks/futures that would release the bytes is the |
||
| if (write.buf.readableBytes() == 0) { | ||
| write.promise.trySuccess(); | ||
| currentWrite = null; | ||
| continue; | ||
| } | ||
| final int readableBytes = write.buf.readableBytes(); | ||
| final int bufferSize = Math.min(readableBytes, 1 << 18); | ||
| final int readerIndex = write.buf.readerIndex(); | ||
| final boolean sliced = readableBytes != bufferSize; | ||
| final ByteBuf writeBuffer; | ||
| if (sliced) { | ||
| writeBuffer = write.buf.retainedSlice(readerIndex, bufferSize); | ||
| write.buf.readerIndex(readerIndex + bufferSize); | ||
| } else { | ||
| writeBuffer = write.buf; | ||
| } | ||
| final ChannelFuture writeFuture = ctx.write(writeBuffer); | ||
| if (sliced == false || write.buf.readableBytes() == 0) { | ||
| currentWrite = null; | ||
| writeFuture.addListener(future -> { | ||
| assert ctx.executor().inEventLoop(); | ||
| if (future.isSuccess()) { | ||
| write.promise.trySuccess(); | ||
| } else { | ||
| write.promise.tryFailure(future.cause()); | ||
| } | ||
| }); | ||
| } else { | ||
| writeFuture.addListener(future -> { | ||
| assert ctx.executor().inEventLoop(); | ||
| if (future.isSuccess() == false) { | ||
| write.promise.tryFailure(future.cause()); | ||
| } | ||
| }); | ||
| } | ||
| ctx.flush(); | ||
| if (channel.isActive() == false) { | ||
| failQueuedWrites(); | ||
| return; | ||
| } | ||
| } | ||
| } | ||
|
|
||
| private void failQueuedWrites() { | ||
| WriteOperation queuedWrite; | ||
| while ((queuedWrite = queuedWrites.poll()) != null) { | ||
| queuedWrite.promise.tryFailure(new ClosedChannelException()); | ||
| } | ||
| } | ||
|
|
||
| private static final class WriteOperation { | ||
|
|
||
| private final ByteBuf buf; | ||
|
|
||
| private final ChannelPromise promise; | ||
|
|
||
| WriteOperation(ByteBuf buf, ChannelPromise promise) { | ||
| this.buf = buf; | ||
| this.promise = promise; | ||
| } | ||
| } | ||
| } | ||
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I'm not sure that this should be in this (
Netty4MessageChannelHandler). You are copying heap bytes -> direct bytes. That works fine for plaintext. But this handler comes before theSslHandler. Which means in the security use case, this is heap bytes -> direct bytes -> heap bytes (temporary byte array inSSLEngine) encrypted-> heap buffer -> direct bytes (infilterOutboundMessages). So this will introduce more copies.I think that either:
SslHandlerThere was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Right :) I went with option 2. now. That also makes the whole issue of potentially trying to write released bytes a lot safer/cleaner. Once/if the buffer is below our write chunk size we can simply pass it down outright and use retained slices in the writes preceding that situation.
(+ we save some objects for writes that are smaller than the buffer size :))