Only use static Exception instances when we can ensure addSuppressed … (#9152)

Motivation:

OOME is occurred by increasing suppressedExceptions because other libraries call Throwable#addSuppressed. As we have no control over what other libraries do we need to ensure this can not lead to OOME.

Modifications:

Only use static instances of the Exceptions if we can either dissable addSuppressed or we run on java6.

Result:

Not possible to OOME because of addSuppressed. Fixes https://github.com/netty/netty/issues/9151.
This commit is contained in:
Norman Maurer 2019-05-17 22:23:02 +02:00
parent 211dde4e08
commit ed61e5f543
28 changed files with 186 additions and 272 deletions

View File

@ -37,7 +37,6 @@ import io.netty.handler.codec.http.HttpResponseDecoder;
import io.netty.handler.codec.http.HttpScheme;
import io.netty.util.NetUtil;
import io.netty.util.ReferenceCountUtil;
import io.netty.util.internal.ThrowableUtil;
import java.net.URI;
import java.nio.channels.ClosedChannelException;
@ -50,8 +49,6 @@ import java.util.concurrent.atomic.AtomicIntegerFieldUpdater;
* Base class for web socket client handshake implementations
*/
public abstract class WebSocketClientHandshaker {
private static final ClosedChannelException CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), WebSocketClientHandshaker.class, "processHandshake(...)");
private static final String HTTP_SCHEME_PREFIX = HttpScheme.HTTP + "://";
private static final String HTTPS_SCHEME_PREFIX = HttpScheme.HTTPS + "://";
@ -421,7 +418,9 @@ public abstract class WebSocketClientHandshaker {
@Override
public void channelInactive(ChannelHandlerContext ctx) throws Exception {
// Fail promise if Channel was closed
promise.tryFailure(CLOSED_CHANNEL_EXCEPTION);
if (!promise.isDone()) {
promise.tryFailure(new ClosedChannelException());
}
ctx.fireChannelInactive();
}
});

View File

@ -49,8 +49,6 @@ import java.util.Set;
*/
public abstract class WebSocketServerHandshaker {
protected static final InternalLogger logger = InternalLoggerFactory.getInstance(WebSocketServerHandshaker.class);
private static final ClosedChannelException CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), WebSocketServerHandshaker.class, "handshake(...)");
private final String uri;
@ -281,7 +279,9 @@ public abstract class WebSocketServerHandshaker {
@Override
public void channelInactive(ChannelHandlerContext ctx) throws Exception {
// Fail promise if Channel was closed
promise.tryFailure(CLOSED_CHANNEL_EXCEPTION);
if (!promise.isDone()) {
promise.tryFailure(new ClosedChannelException());
}
ctx.fireChannelInactive();
}
});

View File

@ -53,24 +53,31 @@ import static io.netty.util.internal.ThrowableUtil.unknownStackTrace;
final class HpackDecoder {
private static final Http2Exception DECODE_ULE_128_DECOMPRESSION_EXCEPTION = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - decompression failure"), HpackDecoder.class,
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - decompression failure",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class,
"decodeULE128(..)");
private static final Http2Exception DECODE_ULE_128_TO_LONG_DECOMPRESSION_EXCEPTION = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - long overflow"), HpackDecoder.class, "decodeULE128(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - long overflow",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "decodeULE128(..)");
private static final Http2Exception DECODE_ULE_128_TO_INT_DECOMPRESSION_EXCEPTION = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - int overflow"), HpackDecoder.class, "decodeULE128ToInt(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - int overflow",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "decodeULE128ToInt(..)");
private static final Http2Exception DECODE_ILLEGAL_INDEX_VALUE = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - illegal index value"), HpackDecoder.class, "decode(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - illegal index value",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "decode(..)");
private static final Http2Exception INDEX_HEADER_ILLEGAL_INDEX_VALUE = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - illegal index value"), HpackDecoder.class, "indexHeader(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - illegal index value",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "indexHeader(..)");
private static final Http2Exception READ_NAME_ILLEGAL_INDEX_VALUE = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - illegal index value"), HpackDecoder.class, "readName(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - illegal index value",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "readName(..)");
private static final Http2Exception INVALID_MAX_DYNAMIC_TABLE_SIZE = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - invalid max dynamic table size"), HpackDecoder.class,
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - invalid max dynamic table size",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class,
"setDynamicTableSize(..)");
private static final Http2Exception MAX_DYNAMIC_TABLE_SIZE_CHANGE_REQUIRED = unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - max dynamic table size change required"), HpackDecoder.class,
"decode(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - max dynamic table size change required",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackDecoder.class, "decode(..)");
private static final byte READ_HEADER_REPRESENTATION = 0;
private static final byte READ_MAX_DYNAMIC_TABLE_SIZE = 1;
private static final byte READ_INDEXED_HEADER = 2;

View File

@ -43,9 +43,11 @@ import static io.netty.handler.codec.http2.Http2Exception.connectionError;
final class HpackHuffmanDecoder {
private static final Http2Exception EOS_DECODED = ThrowableUtil.unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - EOS Decoded"), HpackHuffmanDecoder.class, "decode(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - EOS Decoded",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackHuffmanDecoder.class, "decode(..)");
private static final Http2Exception INVALID_PADDING = ThrowableUtil.unknownStackTrace(
connectionError(COMPRESSION_ERROR, "HPACK - Invalid Padding"), HpackHuffmanDecoder.class, "decode(..)");
Http2Exception.newStatic(COMPRESSION_ERROR, "HPACK - Invalid Padding",
Http2Exception.ShutdownHint.HARD_SHUTDOWN), HpackHuffmanDecoder.class, "decode(..)");
private static final Node ROOT = buildTree(HpackUtil.HUFFMAN_CODES, HpackUtil.HUFFMAN_CODE_LENGTHS);

View File

@ -15,6 +15,7 @@
package io.netty.handler.codec.http2;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.UnstableApi;
import java.util.ArrayList;
@ -62,6 +63,20 @@ public class Http2Exception extends Exception {
this.shutdownHint = requireNonNull(shutdownHint, "shutdownHint");
}
static Http2Exception newStatic(Http2Error error, String message, ShutdownHint shutdownHint) {
if (PlatformDependent.javaVersion() >= 7) {
return new Http2Exception(error, message, shutdownHint, true);
}
return new Http2Exception(error, message, shutdownHint);
}
private Http2Exception(Http2Error error, String message, ShutdownHint shutdownHint, boolean shared) {
super(message, null, false, true);
assert shared;
this.error = requireNonNull(error, "error");
this.shutdownHint = requireNonNull(shutdownHint, "shutdownHint");
}
public Http2Error error() {
return error;
}

View File

@ -112,8 +112,6 @@ public class Http2MultiplexCodec extends Http2FrameCodec {
private static final ChannelFutureListener CHILD_CHANNEL_REGISTRATION_LISTENER = Http2MultiplexCodec::registerDone;
private static final ChannelMetadata METADATA = new ChannelMetadata(false, 16);
private static final ClosedChannelException CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), DefaultHttp2StreamChannel.Http2ChannelUnsafe.class, "write(...)");
/**
* Number of bytes to consider non-payload messages. 9 is arbitrary, but also the minimum size of an HTTP/2 frame.
* Primarily is non-zero.
@ -1087,7 +1085,7 @@ public class Http2MultiplexCodec extends Http2FrameCodec {
// Once the outbound side was closed we should not allow header / data frames
outboundClosed && (msg instanceof Http2HeadersFrame || msg instanceof Http2DataFrame)) {
ReferenceCountUtil.release(msg);
promise.setFailure(CLOSED_CHANNEL_EXCEPTION);
promise.setFailure(new ClosedChannelException());
return;
}

View File

@ -50,7 +50,7 @@ import static io.netty.handler.codec.compression.Lz4Constants.MAGIC_NUMBER;
import static io.netty.handler.codec.compression.Lz4Constants.MAX_BLOCK_SIZE;
import static io.netty.handler.codec.compression.Lz4Constants.MIN_BLOCK_SIZE;
import static io.netty.handler.codec.compression.Lz4Constants.TOKEN_OFFSET;
import static io.netty.util.internal.ThrowableUtil.unknownStackTrace;
import static java.util.Objects.requireNonNull;
/**
@ -70,9 +70,6 @@ import static java.util.Objects.requireNonNull;
* * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * * *
*/
public class Lz4FrameEncoder extends MessageToByteEncoder<ByteBuf> {
private static final EncoderException ENCODE_FINSHED_EXCEPTION = unknownStackTrace(new EncoderException(
new IllegalStateException("encode finished and not enough space to write remaining data")),
Lz4FrameEncoder.class, "encode");
static final int DEFAULT_MAX_ENCODE_SIZE = Integer.MAX_VALUE;
private final int blockSize;
@ -243,7 +240,7 @@ public class Lz4FrameEncoder extends MessageToByteEncoder<ByteBuf> {
if (finished) {
if (!out.isWritable(in.readableBytes())) {
// out should be EMPTY_BUFFER because we should have allocated enough space above in allocateBuffer.
throw ENCODE_FINSHED_EXCEPTION;
throw new IllegalStateException("encode finished and not enough space to write remaining data");
}
out.writeBytes(in);
return;

View File

@ -42,8 +42,6 @@ public class DefaultPromise<V> implements Promise<V> {
AtomicReferenceFieldUpdater.newUpdater(DefaultPromise.class, Object.class, "result");
private static final Object SUCCESS = new Object();
private static final Object UNCANCELLABLE = new Object();
private static final CauseHolder CANCELLATION_CAUSE_HOLDER = new CauseHolder(ThrowableUtil.unknownStackTrace(
new CancellationException(), DefaultPromise.class, "cancel(...)"));
private volatile Object result;
private final EventExecutor executor;
@ -297,7 +295,8 @@ public class DefaultPromise<V> implements Promise<V> {
*/
@Override
public boolean cancel(boolean mayInterruptIfRunning) {
if (RESULT_UPDATER.compareAndSet(this, null, CANCELLATION_CAUSE_HOLDER)) {
if (RESULT_UPDATER.get(this) == null &&
RESULT_UPDATER.compareAndSet(this, null, new CauseHolder(new CancellationException()))) {
if (checkNotifyWaiters()) {
notifyListeners();
}

View File

@ -25,7 +25,7 @@ import java.lang.annotation.Target;
* Annotation to suppress the Java 6 source code requirement checks for a method.
*/
@Retention(RetentionPolicy.CLASS)
@Target({ ElementType.METHOD })
@Target({ ElementType.METHOD, ElementType.CONSTRUCTOR })
public @interface SuppressJava6Requirement {
String reason();

View File

@ -28,7 +28,6 @@ import io.netty.util.ResourceLeakTracker;
import io.netty.util.internal.EmptyArrays;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.StringUtil;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.UnstableApi;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -98,12 +97,6 @@ public class ReferenceCountedOpenSslEngine extends SSLEngine implements Referenc
private static final InternalLogger logger = InternalLoggerFactory.getInstance(ReferenceCountedOpenSslEngine.class);
private static final SSLException BEGIN_HANDSHAKE_ENGINE_CLOSED = ThrowableUtil.unknownStackTrace(
new SSLException("engine closed"), ReferenceCountedOpenSslEngine.class, "beginHandshake()");
private static final SSLException HANDSHAKE_ENGINE_CLOSED = ThrowableUtil.unknownStackTrace(
new SSLException("engine closed"), ReferenceCountedOpenSslEngine.class, "handshake()");
private static final SSLException RENEGOTIATION_UNSUPPORTED = ThrowableUtil.unknownStackTrace(
new SSLException("renegotiation unsupported"), ReferenceCountedOpenSslEngine.class, "beginHandshake()");
private static final ResourceLeakDetector<ReferenceCountedOpenSslEngine> leakDetector =
ResourceLeakDetectorFactory.instance().newResourceLeakDetector(ReferenceCountedOpenSslEngine.class);
private static final int OPENSSL_OP_NO_PROTOCOL_INDEX_SSLV2 = 0;
@ -1638,7 +1631,7 @@ public class ReferenceCountedOpenSslEngine extends SSLEngine implements Referenc
public final synchronized void beginHandshake() throws SSLException {
switch (handshakeState) {
case STARTED_IMPLICITLY:
checkEngineClosed(BEGIN_HANDSHAKE_ENGINE_CLOSED);
checkEngineClosed();
// A user did not start handshake by calling this method by him/herself,
// but handshake has been started already by wrap() or unwrap() implicitly.
@ -1654,7 +1647,7 @@ public class ReferenceCountedOpenSslEngine extends SSLEngine implements Referenc
// Nothing to do as the handshake is not done yet.
break;
case FINISHED:
throw RENEGOTIATION_UNSUPPORTED;
throw new SSLException("renegotiation unsupported");
case NOT_STARTED:
handshakeState = HandshakeState.STARTED_EXPLICITLY;
if (handshake() == NEED_TASK) {
@ -1668,9 +1661,9 @@ public class ReferenceCountedOpenSslEngine extends SSLEngine implements Referenc
}
}
private void checkEngineClosed(SSLException cause) throws SSLException {
private void checkEngineClosed() throws SSLException {
if (isDestroyed()) {
throw cause;
throw new SSLException("engine closed");
}
}
@ -1722,7 +1715,7 @@ public class ReferenceCountedOpenSslEngine extends SSLEngine implements Referenc
return FINISHED;
}
checkEngineClosed(HANDSHAKE_ENGINE_CLOSED);
checkEngineClosed();
if (handshakeException != null) {
return handshakeException();

View File

@ -45,7 +45,6 @@ import io.netty.util.concurrent.ImmediateExecutor;
import io.netty.util.concurrent.Promise;
import io.netty.util.concurrent.PromiseNotifier;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.UnstableApi;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -174,18 +173,6 @@ public class SslHandler extends ByteToMessageDecoder {
private static final Pattern IGNORABLE_ERROR_MESSAGE = Pattern.compile(
"^.*(?:connection.*(?:reset|closed|abort|broken)|broken.*pipe).*$", Pattern.CASE_INSENSITIVE);
/**
* Used in {@link #unwrapNonAppData(ChannelHandlerContext)} as input for
* {@link #unwrap(ChannelHandlerContext, ByteBuf, int, int)}. Using this static instance reduce object
* creation as {@link Unpooled#EMPTY_BUFFER#nioBuffer()} creates a new {@link ByteBuffer} everytime.
*/
private static final SSLException SSLENGINE_CLOSED = ThrowableUtil.unknownStackTrace(
new SSLException("SSLEngine closed already"), SslHandler.class, "wrap(...)");
private static final SSLException HANDSHAKE_TIMED_OUT = ThrowableUtil.unknownStackTrace(
new SSLException("handshake timed out"), SslHandler.class, "handshake(...)");
private static final ClosedChannelException CHANNEL_CLOSED = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), SslHandler.class, "channelInactive(...)");
/**
* <a href="https://tools.ietf.org/html/rfc5246#section-6.2">2^14</a> which is the maximum sized plaintext chunk
* allowed by the TLS RFC.
@ -839,11 +826,12 @@ public class SslHandler extends ByteToMessageDecoder {
if (result.getStatus() == Status.CLOSED) {
buf.release();
buf = null;
promise.tryFailure(SSLENGINE_CLOSED);
SSLException exception = new SSLException("SSLEngine closed already");
promise.tryFailure(exception);
promise = null;
// SSLEngine has been closed already.
// Any further write attempts should be denied.
pendingUnencryptedWrites.releaseAndFailAll(ctx, SSLENGINE_CLOSED);
pendingUnencryptedWrites.releaseAndFailAll(ctx, exception);
return;
} else {
if (buf.isReadable()) {
@ -1063,12 +1051,13 @@ public class SslHandler extends ByteToMessageDecoder {
@Override
public void channelInactive(ChannelHandlerContext ctx) throws Exception {
ClosedChannelException exception = new ClosedChannelException();
// Make sure to release SSLEngine,
// and notify the handshake future if the connection has been closed during handshake.
setHandshakeFailure(ctx, CHANNEL_CLOSED, !outboundClosed, handshakeStarted, false);
setHandshakeFailure(ctx, exception, !outboundClosed, handshakeStarted, false);
// Ensure we always notify the sslClosePromise as well
notifyClosePromise(CHANNEL_CLOSED);
notifyClosePromise(exception);
super.channelInactive(ctx);
}
@ -1960,12 +1949,14 @@ public class SslHandler extends ByteToMessageDecoder {
if (localHandshakePromise.isDone()) {
return;
}
SSLException exception = new SSLException("handshake timed out");
try {
if (localHandshakePromise.tryFailure(HANDSHAKE_TIMED_OUT)) {
SslUtils.handleHandshakeFailure(ctx, HANDSHAKE_TIMED_OUT, true);
if (localHandshakePromise.tryFailure(exception)) {
SslUtils.handleHandshakeFailure(ctx, exception, true);
}
} finally {
releaseAndFailAll(HANDSHAKE_TIMED_OUT);
releaseAndFailAll(exception);
}
}, handshakeTimeoutMillis, TimeUnit.MILLISECONDS);

View File

@ -22,7 +22,6 @@ import io.netty.channel.ChannelInboundHandler;
import io.netty.handler.ssl.ReferenceCountedOpenSslContext;
import io.netty.handler.ssl.ReferenceCountedOpenSslEngine;
import io.netty.handler.ssl.SslHandshakeCompletionEvent;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.UnstableApi;
import javax.net.ssl.SSLHandshakeException;
@ -36,9 +35,6 @@ import javax.net.ssl.SSLHandshakeException;
@UnstableApi
public abstract class OcspClientHandler implements ChannelInboundHandler {
private static final SSLHandshakeException OCSP_VERIFICATION_EXCEPTION = ThrowableUtil.unknownStackTrace(
new SSLHandshakeException("Bad OCSP response"), OcspClientHandler.class, "verify(...)");
private final ReferenceCountedOpenSslEngine engine;
protected OcspClientHandler(ReferenceCountedOpenSslEngine engine) {
@ -57,7 +53,7 @@ public abstract class OcspClientHandler implements ChannelInboundHandler {
SslHandshakeCompletionEvent event = (SslHandshakeCompletionEvent) evt;
if (event.isSuccess() && !verify(ctx, engine)) {
throw OCSP_VERIFICATION_EXCEPTION;
throw new SSLHandshakeException("Bad OCSP response");
}
}

View File

@ -25,5 +25,7 @@ public final class ReadTimeoutException extends TimeoutException {
public static final ReadTimeoutException INSTANCE = new ReadTimeoutException();
private ReadTimeoutException() { }
private ReadTimeoutException() {
super(true);
}
}

View File

@ -25,7 +25,12 @@ public class TimeoutException extends ChannelException {
private static final long serialVersionUID = 4673641882869672533L;
TimeoutException() { }
TimeoutException() {
}
TimeoutException(boolean shared) {
super(null, null, shared);
}
@Override
public Throwable fillInStackTrace() {

View File

@ -25,5 +25,8 @@ public final class WriteTimeoutException extends TimeoutException {
public static final WriteTimeoutException INSTANCE = new WriteTimeoutException();
private WriteTimeoutException() { }
private WriteTimeoutException() {
super(true);
}
}

View File

@ -63,15 +63,15 @@ import static java.util.Objects.requireNonNull;
abstract class DnsResolveContext<T> {
private static final RuntimeException NXDOMAIN_QUERY_FAILED_EXCEPTION = ThrowableUtil.unknownStackTrace(
new RuntimeException("No answer found and NXDOMAIN response code returned"),
DnsResolveContextException.newStatic("No answer found and NXDOMAIN response code returned"),
DnsResolveContext.class,
"onResponse(..)");
private static final RuntimeException CNAME_NOT_FOUND_QUERY_FAILED_EXCEPTION = ThrowableUtil.unknownStackTrace(
new RuntimeException("No matching CNAME record found"),
DnsResolveContextException.newStatic("No matching CNAME record found"),
DnsResolveContext.class,
"onResponseCNAME(..)");
private static final RuntimeException NO_MATCHING_RECORD_QUERY_FAILED_EXCEPTION = ThrowableUtil.unknownStackTrace(
new RuntimeException("No matching record type found"),
DnsResolveContextException.newStatic("No matching record type found"),
DnsResolveContext.class,
"onResponseAorAAAA(..)");
private static final RuntimeException UNRECOGNIZED_TYPE_QUERY_FAILED_EXCEPTION = ThrowableUtil.unknownStackTrace(
@ -79,7 +79,7 @@ abstract class DnsResolveContext<T> {
DnsResolveContext.class,
"onResponse(..)");
private static final RuntimeException NAME_SERVERS_EXHAUSTED_EXCEPTION = ThrowableUtil.unknownStackTrace(
new RuntimeException("No name servers returned an answer"),
DnsResolveContextException.newStatic("No name servers returned an answer"),
DnsResolveContext.class,
"tryToFinishResolve(..)");
@ -117,6 +117,17 @@ abstract class DnsResolveContext<T> {
allowedQueries = maxAllowedQueries;
}
static final class DnsResolveContextException extends RuntimeException {
private DnsResolveContextException(String message) {
super(message, null, false, true);
}
static DnsResolveContextException newStatic(String message) {
return new DnsResolveContextException(message);
}
}
/**
* The {@link DnsCache} to use while resolving.
*/

View File

@ -56,8 +56,6 @@ import static io.netty.channel.unix.UnixChannelUtil.computeRemoteAddr;
import static java.util.Objects.requireNonNull;
abstract class AbstractEpollChannel extends AbstractChannel implements UnixChannel {
private static final ClosedChannelException DO_CLOSE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), AbstractEpollChannel.class, "doClose()");
private static final ChannelMetadata METADATA = new ChannelMetadata(false);
final LinuxSocket socket;
/**
@ -163,7 +161,7 @@ abstract class AbstractEpollChannel extends AbstractChannel implements UnixChann
ChannelPromise promise = connectPromise;
if (promise != null) {
// Use tryFailure() instead of setFailure() to avoid the race against cancel().
promise.tryFailure(DO_CLOSE_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
connectPromise = null;
}

View File

@ -37,7 +37,6 @@ import io.netty.channel.unix.SocketWritableByteChannel;
import io.netty.channel.unix.UnixChannelUtil;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.StringUtil;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.UnstableApi;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -62,15 +61,6 @@ public abstract class AbstractEpollStreamChannel extends AbstractEpollChannel im
" (expected: " + StringUtil.simpleClassName(ByteBuf.class) + ", " +
StringUtil.simpleClassName(DefaultFileRegion.class) + ')';
private static final InternalLogger logger = InternalLoggerFactory.getInstance(AbstractEpollStreamChannel.class);
private static final ClosedChannelException CLEAR_SPLICE_QUEUE_CLOSED_CHANNEL_EXCEPTION =
ThrowableUtil.unknownStackTrace(new ClosedChannelException(),
AbstractEpollStreamChannel.class, "clearSpliceQueue()");
private static final ClosedChannelException SPLICE_TO_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(),
AbstractEpollStreamChannel.class, "spliceTo(...)");
private static final ClosedChannelException FAIL_SPLICE_IF_CLOSED_CLOSED_CHANNEL_EXCEPTION =
ThrowableUtil.unknownStackTrace(new ClosedChannelException(),
AbstractEpollStreamChannel.class, "failSpliceIfClosed(...)");
private final Runnable flushTask = () -> {
// Calling flush0 directly to ensure we not try to flush messages that were added via write(...) in the
// meantime.
@ -168,7 +158,7 @@ public abstract class AbstractEpollStreamChannel extends AbstractEpollChannel im
}
requireNonNull(promise, "promise");
if (!isOpen()) {
promise.tryFailure(SPLICE_TO_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
} else {
addToSpliceQueue(new SpliceInChannelTask(ch, len, promise));
failSpliceIfClosed(promise);
@ -217,7 +207,7 @@ public abstract class AbstractEpollStreamChannel extends AbstractEpollChannel im
}
requireNonNull(promise, "promise");
if (!isOpen()) {
promise.tryFailure(SPLICE_TO_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
} else {
addToSpliceQueue(new SpliceFdTask(ch, offset, len, promise));
failSpliceIfClosed(promise);
@ -229,7 +219,7 @@ public abstract class AbstractEpollStreamChannel extends AbstractEpollChannel im
if (!isOpen()) {
// Seems like the Channel was closed in the meantime try to fail the promise to prevent any
// cases where a future may not be notified otherwise.
if (promise.tryFailure(FAIL_SPLICE_IF_CLOSED_CLOSED_CHANNEL_EXCEPTION)) {
if (promise.tryFailure(new ClosedChannelException())) {
// Call this via the EventLoop as it is a MPSC queue.
eventLoop().execute(this::clearSpliceQueue);
}
@ -659,12 +649,17 @@ public abstract class AbstractEpollStreamChannel extends AbstractEpollChannel im
if (spliceQueue == null) {
return;
}
ClosedChannelException exception = null;
for (;;) {
SpliceInTask task = spliceQueue.poll();
if (task == null) {
break;
}
task.promise.tryFailure(CLEAR_SPLICE_QUEUE_CLOSED_CHANNEL_EXCEPTION);
if (exception == null) {
exception = new ClosedChannelException();
}
task.promise.tryFailure(exception);
}
}

View File

@ -16,29 +16,20 @@
package io.netty.channel.epoll;
import io.netty.channel.DefaultFileRegion;
import io.netty.channel.unix.Errors.NativeIoException;
import io.netty.channel.unix.NativeInetAddress;
import io.netty.channel.unix.PeerCredentials;
import io.netty.channel.unix.Socket;
import io.netty.util.internal.ThrowableUtil;
import java.io.IOException;
import java.net.InetAddress;
import java.nio.channels.ClosedChannelException;
import static io.netty.channel.unix.Errors.ERRNO_EPIPE_NEGATIVE;
import static io.netty.channel.unix.Errors.ioResult;
import static io.netty.channel.unix.Errors.newConnectionResetException;
/**
* A socket which provides access Linux native methods.
*/
final class LinuxSocket extends Socket {
private static final long MAX_UINT32_T = 0xFFFFFFFFL;
private static final NativeIoException SENDFILE_CONNECTION_RESET_EXCEPTION =
newConnectionResetException("syscall:sendfile(...)", ERRNO_EPIPE_NEGATIVE);
private static final ClosedChannelException SENDFILE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), Native.class, "sendfile(...)");
LinuxSocket(int fd) {
super(fd);
@ -177,7 +168,7 @@ final class LinuxSocket extends Socket {
if (res >= 0) {
return res;
}
return ioResult("sendfile", (int) res, SENDFILE_CONNECTION_RESET_EXCEPTION, SENDFILE_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendfile", (int) res);
}
public static LinuxSocket newSocketStream() {

View File

@ -15,7 +15,6 @@
*/
package io.netty.channel.epoll;
import io.netty.channel.unix.Errors.NativeIoException;
import io.netty.channel.unix.FileDescriptor;
import io.netty.channel.unix.Socket;
import io.netty.util.internal.NativeLibraryLoader;
@ -26,7 +25,6 @@ import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
import java.io.IOException;
import java.nio.channels.ClosedChannelException;
import java.util.Locale;
import static io.netty.channel.epoll.NativeStaticallyReferencedJniMethods.epollerr;
@ -38,9 +36,7 @@ import static io.netty.channel.epoll.NativeStaticallyReferencedJniMethods.isSupp
import static io.netty.channel.epoll.NativeStaticallyReferencedJniMethods.isSupportingTcpFastopen;
import static io.netty.channel.epoll.NativeStaticallyReferencedJniMethods.kernelVersion;
import static io.netty.channel.epoll.NativeStaticallyReferencedJniMethods.tcpMd5SigMaxKeyLen;
import static io.netty.channel.unix.Errors.ERRNO_EPIPE_NEGATIVE;
import static io.netty.channel.unix.Errors.ioResult;
import static io.netty.channel.unix.Errors.newConnectionResetException;
import static io.netty.channel.unix.Errors.newIOException;
/**
@ -75,20 +71,6 @@ public final class Native {
public static final int TCP_MD5SIG_MAXKEYLEN = tcpMd5SigMaxKeyLen();
public static final String KERNEL_VERSION = kernelVersion();
private static final NativeIoException SENDMMSG_CONNECTION_RESET_EXCEPTION;
private static final NativeIoException SPLICE_CONNECTION_RESET_EXCEPTION;
private static final ClosedChannelException SENDMMSG_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), Native.class, "sendmmsg(...)");
private static final ClosedChannelException SPLICE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), Native.class, "splice(...)");
static {
SENDMMSG_CONNECTION_RESET_EXCEPTION = newConnectionResetException("syscall:sendmmsg(...)",
ERRNO_EPIPE_NEGATIVE);
SPLICE_CONNECTION_RESET_EXCEPTION = newConnectionResetException("syscall:splice(...)",
ERRNO_EPIPE_NEGATIVE);
}
public static FileDescriptor newEventFd() {
return new FileDescriptor(eventFd());
}
@ -165,7 +147,7 @@ public final class Native {
if (res >= 0) {
return res;
}
return ioResult("splice", res, SPLICE_CONNECTION_RESET_EXCEPTION, SPLICE_CLOSED_CHANNEL_EXCEPTION);
return ioResult("splice", res);
}
private static native int splice0(int fd, long offIn, int fdOut, long offOut, long len);
@ -176,7 +158,7 @@ public final class Native {
if (res >= 0) {
return res;
}
return ioResult("sendmmsg", res, SENDMMSG_CONNECTION_RESET_EXCEPTION, SENDMMSG_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendmmsg", res);
}
private static native int sendmmsg0(

View File

@ -16,26 +16,18 @@
package io.netty.channel.kqueue;
import io.netty.channel.DefaultFileRegion;
import io.netty.channel.unix.Errors;
import io.netty.channel.unix.PeerCredentials;
import io.netty.channel.unix.Socket;
import io.netty.util.internal.ThrowableUtil;
import java.io.IOException;
import java.nio.channels.ClosedChannelException;
import static io.netty.channel.kqueue.AcceptFilter.PLATFORM_UNSUPPORTED;
import static io.netty.channel.unix.Errors.ERRNO_EPIPE_NEGATIVE;
import static io.netty.channel.unix.Errors.ioResult;
import static io.netty.channel.unix.Errors.newConnectionResetException;
/**
* A socket which provides access BSD native methods.
*/
final class BsdSocket extends Socket {
private static final Errors.NativeIoException SENDFILE_CONNECTION_RESET_EXCEPTION;
private static final ClosedChannelException SENDFILE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), Native.class, "sendfile(..)");
// These limits are just based on observations. I couldn't find anything in header files which formally
// define these limits.
@ -43,11 +35,6 @@ final class BsdSocket extends Socket {
private static final int FREEBSD_SND_LOW_AT_MAX = 1 << 15;
static final int BSD_SND_LOW_AT_MAX = Math.min(APPLE_SND_LOW_AT_MAX, FREEBSD_SND_LOW_AT_MAX);
static {
SENDFILE_CONNECTION_RESET_EXCEPTION = newConnectionResetException("syscall:sendfile",
ERRNO_EPIPE_NEGATIVE);
}
BsdSocket(int fd) {
super(fd);
}
@ -90,7 +77,7 @@ final class BsdSocket extends Socket {
if (res >= 0) {
return res;
}
return ioResult("sendfile", (int) res, SENDFILE_CONNECTION_RESET_EXCEPTION, SENDFILE_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendfile", (int) res);
}
public static BsdSocket newSocketStream() {

View File

@ -62,14 +62,29 @@ public final class Errors {
public static final class NativeIoException extends IOException {
private static final long serialVersionUID = 8222160204268655526L;
private final int expectedErr;
private final boolean fillInStackTrace;
public NativeIoException(String method, int expectedErr) {
this(method, expectedErr, true);
}
public NativeIoException(String method, int expectedErr, boolean fillInStackTrace) {
super(method + "(..) failed: " + ERRORS[-expectedErr]);
this.expectedErr = expectedErr;
this.fillInStackTrace = fillInStackTrace;
}
public int expectedErr() {
return expectedErr;
}
@Override
public synchronized Throwable fillInStackTrace() {
if (fillInStackTrace) {
return super.fillInStackTrace();
}
return this;
}
}
static final class NativeConnectException extends ConnectException {
@ -92,11 +107,8 @@ public final class Errors {
}
}
static void throwConnectException(String method, NativeConnectException refusedCause, int err)
static void throwConnectException(String method, int err)
throws IOException {
if (err == refusedCause.expectedErr()) {
throw refusedCause;
}
if (err == ERROR_EALREADY_NEGATIVE) {
throw new ConnectionPendingException();
}
@ -113,7 +125,7 @@ public final class Errors {
}
public static NativeIoException newConnectionResetException(String method, int errnoNegative) {
NativeIoException exception = newIOException(method, errnoNegative);
NativeIoException exception = new NativeIoException(method, errnoNegative, false);
exception.setStackTrace(EmptyArrays.EMPTY_STACK_TRACE);
return exception;
}
@ -122,6 +134,7 @@ public final class Errors {
return new NativeIoException(method, err);
}
@Deprecated
public static int ioResult(String method, int err, NativeIoException resetCause,
ClosedChannelException closedCause) throws IOException {
// network stack saturated... try again later
@ -146,5 +159,23 @@ public final class Errors {
throw newIOException(method, err);
}
public static int ioResult(String method, int err) throws IOException {
// network stack saturated... try again later
if (err == ERRNO_EAGAIN_NEGATIVE || err == ERRNO_EWOULDBLOCK_NEGATIVE) {
return 0;
}
if (err == ERRNO_EBADF_NEGATIVE) {
throw new ClosedChannelException();
}
if (err == ERRNO_ENOTCONN_NEGATIVE) {
throw new NotYetConnectedException();
}
if (err == ERRNO_ENOENT_NEGATIVE) {
throw new FileNotFoundException();
}
throw new NativeIoException(method, err, false);
}
private Errors() { }
}

View File

@ -15,12 +15,10 @@
*/
package io.netty.channel.unix;
import io.netty.util.internal.ThrowableUtil;
import java.io.File;
import java.io.IOException;
import java.nio.ByteBuffer;
import java.nio.channels.ClosedChannelException;
import java.util.concurrent.atomic.AtomicIntegerFieldUpdater;
import static io.netty.channel.unix.Errors.ioResult;
@ -35,36 +33,6 @@ import static java.util.Objects.requireNonNull;
* {@link FileDescriptor} for it.
*/
public class FileDescriptor {
private static final ClosedChannelException WRITE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), FileDescriptor.class, "write(..)");
private static final ClosedChannelException WRITE_ADDRESS_CLOSED_CHANNEL_EXCEPTION =
ThrowableUtil.unknownStackTrace(new ClosedChannelException(), FileDescriptor.class, "writeAddress(..)");
private static final ClosedChannelException WRITEV_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), FileDescriptor.class, "writev(..)");
private static final ClosedChannelException WRITEV_ADDRESSES_CLOSED_CHANNEL_EXCEPTION =
ThrowableUtil.unknownStackTrace(new ClosedChannelException(), FileDescriptor.class, "writevAddresses(..)");
private static final ClosedChannelException READ_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), FileDescriptor.class, "read(..)");
private static final ClosedChannelException READ_ADDRESS_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), FileDescriptor.class, "readAddress(..)");
private static final Errors.NativeIoException WRITE_CONNECTION_RESET_EXCEPTION = ThrowableUtil.unknownStackTrace(
Errors.newConnectionResetException("syscall:write", Errors.ERRNO_EPIPE_NEGATIVE),
FileDescriptor.class, "write(..)");
private static final Errors.NativeIoException WRITE_ADDRESS_CONNECTION_RESET_EXCEPTION =
ThrowableUtil.unknownStackTrace(Errors.newConnectionResetException("syscall:write",
Errors.ERRNO_EPIPE_NEGATIVE), FileDescriptor.class, "writeAddress(..)");
private static final Errors.NativeIoException WRITEV_CONNECTION_RESET_EXCEPTION = ThrowableUtil.unknownStackTrace(
Errors.newConnectionResetException("syscall:writev", Errors.ERRNO_EPIPE_NEGATIVE),
FileDescriptor.class, "writev(..)");
private static final Errors.NativeIoException WRITEV_ADDRESSES_CONNECTION_RESET_EXCEPTION =
ThrowableUtil.unknownStackTrace(Errors.newConnectionResetException("syscall:writev",
Errors.ERRNO_EPIPE_NEGATIVE), FileDescriptor.class, "writeAddresses(..)");
private static final Errors.NativeIoException READ_CONNECTION_RESET_EXCEPTION = ThrowableUtil.unknownStackTrace(
Errors.newConnectionResetException("syscall:read", Errors.ERRNO_ECONNRESET_NEGATIVE),
FileDescriptor.class, "read(..)");
private static final Errors.NativeIoException READ_ADDRESS_CONNECTION_RESET_EXCEPTION =
ThrowableUtil.unknownStackTrace(Errors.newConnectionResetException("syscall:read",
Errors.ERRNO_ECONNRESET_NEGATIVE), FileDescriptor.class, "readAddress(..)");
private static final AtomicIntegerFieldUpdater<FileDescriptor> stateUpdater =
AtomicIntegerFieldUpdater.newUpdater(FileDescriptor.class, "state");
@ -126,7 +94,7 @@ public class FileDescriptor {
if (res >= 0) {
return res;
}
return ioResult("write", res, WRITE_CONNECTION_RESET_EXCEPTION, WRITE_CLOSED_CHANNEL_EXCEPTION);
return ioResult("write", res);
}
public final int writeAddress(long address, int pos, int limit) throws IOException {
@ -134,8 +102,7 @@ public class FileDescriptor {
if (res >= 0) {
return res;
}
return ioResult("writeAddress", res,
WRITE_ADDRESS_CONNECTION_RESET_EXCEPTION, WRITE_ADDRESS_CLOSED_CHANNEL_EXCEPTION);
return ioResult("writeAddress", res);
}
public final long writev(ByteBuffer[] buffers, int offset, int length, long maxBytesToWrite) throws IOException {
@ -143,7 +110,7 @@ public class FileDescriptor {
if (res >= 0) {
return res;
}
return ioResult("writev", (int) res, WRITEV_CONNECTION_RESET_EXCEPTION, WRITEV_CLOSED_CHANNEL_EXCEPTION);
return ioResult("writev", (int) res);
}
public final long writevAddresses(long memoryAddress, int length) throws IOException {
@ -151,8 +118,7 @@ public class FileDescriptor {
if (res >= 0) {
return res;
}
return ioResult("writevAddresses", (int) res,
WRITEV_ADDRESSES_CONNECTION_RESET_EXCEPTION, WRITEV_ADDRESSES_CLOSED_CHANNEL_EXCEPTION);
return ioResult("writevAddresses", (int) res);
}
public final int read(ByteBuffer buf, int pos, int limit) throws IOException {
@ -163,7 +129,7 @@ public class FileDescriptor {
if (res == 0) {
return -1;
}
return ioResult("read", res, READ_CONNECTION_RESET_EXCEPTION, READ_CLOSED_CHANNEL_EXCEPTION);
return ioResult("read", res);
}
public final int readAddress(long address, int pos, int limit) throws IOException {
@ -174,8 +140,7 @@ public class FileDescriptor {
if (res == 0) {
return -1;
}
return ioResult("readAddress", res,
READ_ADDRESS_CONNECTION_RESET_EXCEPTION, READ_ADDRESS_CLOSED_CHANNEL_EXCEPTION);
return ioResult("readAddress", res);
}
@Override

View File

@ -39,39 +39,12 @@ import static io.netty.channel.unix.Errors.throwConnectException;
import static io.netty.channel.unix.LimitsStaticallyReferencedJniMethods.udsSunPathSize;
import static io.netty.channel.unix.NativeInetAddress.address;
import static io.netty.channel.unix.NativeInetAddress.ipv4MappedIpv6Address;
import static io.netty.util.internal.ThrowableUtil.unknownStackTrace;
/**
* Provides a JNI bridge to native socket operations.
* <strong>Internal usage only!</strong>
*/
public class Socket extends FileDescriptor {
private static final ClosedChannelException SHUTDOWN_CLOSED_CHANNEL_EXCEPTION = unknownStackTrace(
new ClosedChannelException(), Socket.class, "shutdown(..)");
private static final ClosedChannelException SEND_TO_CLOSED_CHANNEL_EXCEPTION = unknownStackTrace(
new ClosedChannelException(), Socket.class, "sendTo(..)");
private static final ClosedChannelException SEND_TO_ADDRESS_CLOSED_CHANNEL_EXCEPTION =
unknownStackTrace(new ClosedChannelException(), Socket.class, "sendToAddress(..)");
private static final ClosedChannelException SEND_TO_ADDRESSES_CLOSED_CHANNEL_EXCEPTION =
unknownStackTrace(new ClosedChannelException(), Socket.class, "sendToAddresses(..)");
private static final Errors.NativeIoException SEND_TO_CONNECTION_RESET_EXCEPTION = unknownStackTrace(
Errors.newConnectionResetException("syscall:sendto", Errors.ERRNO_EPIPE_NEGATIVE),
Socket.class, "sendTo(..)");
private static final Errors.NativeIoException SEND_TO_ADDRESS_CONNECTION_RESET_EXCEPTION =
unknownStackTrace(Errors.newConnectionResetException("syscall:sendto",
Errors.ERRNO_EPIPE_NEGATIVE), Socket.class, "sendToAddress");
private static final Errors.NativeIoException CONNECTION_RESET_EXCEPTION_SENDMSG = unknownStackTrace(
Errors.newConnectionResetException("syscall:sendmsg",
Errors.ERRNO_EPIPE_NEGATIVE), Socket.class, "sendToAddresses(..)");
private static final Errors.NativeIoException CONNECTION_RESET_SHUTDOWN_EXCEPTION =
unknownStackTrace(Errors.newConnectionResetException("syscall:shutdown",
Errors.ERRNO_ECONNRESET_NEGATIVE), Socket.class, "shutdown");
private static final Errors.NativeConnectException FINISH_CONNECT_REFUSED_EXCEPTION =
unknownStackTrace(new Errors.NativeConnectException("syscall:getsockopt",
Errors.ERROR_ECONNREFUSED_NEGATIVE), Socket.class, "finishConnect(..)");
private static final Errors.NativeConnectException CONNECT_REFUSED_EXCEPTION =
unknownStackTrace(new Errors.NativeConnectException("syscall:connect",
Errors.ERROR_ECONNREFUSED_NEGATIVE), Socket.class, "connect(..)");
public static final int UDS_SUN_PATH_SIZE = udsSunPathSize();
@ -111,7 +84,7 @@ public class Socket extends FileDescriptor {
}
int res = shutdown(fd, read, write);
if (res < 0) {
ioResult("shutdown", res, CONNECTION_RESET_SHUTDOWN_EXCEPTION, SHUTDOWN_CLOSED_CHANNEL_EXCEPTION);
ioResult("shutdown", res);
}
}
@ -148,7 +121,7 @@ public class Socket extends FileDescriptor {
if (res == ERROR_ECONNREFUSED_NEGATIVE) {
throw new PortUnreachableException("sendTo failed");
}
return ioResult("sendTo", res, SEND_TO_CONNECTION_RESET_EXCEPTION, SEND_TO_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendTo", res);
}
public final int sendToAddress(long memoryAddress, int pos, int limit, InetAddress addr, int port)
@ -172,8 +145,7 @@ public class Socket extends FileDescriptor {
if (res == ERROR_ECONNREFUSED_NEGATIVE) {
throw new PortUnreachableException("sendToAddress failed");
}
return ioResult("sendToAddress", res,
SEND_TO_ADDRESS_CONNECTION_RESET_EXCEPTION, SEND_TO_ADDRESS_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendToAddress", res);
}
public final int sendToAddresses(long memoryAddress, int length, InetAddress addr, int port) throws IOException {
@ -197,8 +169,7 @@ public class Socket extends FileDescriptor {
if (res == ERROR_ECONNREFUSED_NEGATIVE) {
throw new PortUnreachableException("sendToAddresses failed");
}
return ioResult("sendToAddresses", res,
CONNECTION_RESET_EXCEPTION_SENDMSG, SEND_TO_ADDRESSES_CLOSED_CHANNEL_EXCEPTION);
return ioResult("sendToAddresses", res);
}
public final DatagramSocketAddress recvFrom(ByteBuffer buf, int pos, int limit) throws IOException {
@ -254,7 +225,7 @@ public class Socket extends FileDescriptor {
// connect not complete yet need to wait for EPOLLOUT event
return false;
}
throwConnectException("connect", CONNECT_REFUSED_EXCEPTION, res);
throwConnectException("connect", res);
}
return true;
}
@ -266,7 +237,7 @@ public class Socket extends FileDescriptor {
// connect still in progress
return false;
}
throwConnectException("finishConnect", FINISH_CONNECT_REFUSED_EXCEPTION, res);
throwConnectException("finishConnect", res);
}
return true;
}
@ -274,7 +245,7 @@ public class Socket extends FileDescriptor {
public final void disconnect() throws IOException {
int res = disconnect(fd);
if (res < 0) {
throwConnectException("disconnect", FINISH_CONNECT_REFUSED_EXCEPTION, res);
throwConnectException("disconnect", res);
}
}

View File

@ -24,7 +24,6 @@ import io.netty.util.DefaultAttributeMap;
import io.netty.util.ReferenceCountUtil;
import io.netty.util.concurrent.EventExecutor;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.UnstableApi;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -47,17 +46,6 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
private static final InternalLogger logger = InternalLoggerFactory.getInstance(AbstractChannel.class);
private static final ClosedChannelException ENSURE_OPEN_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(null), AbstractUnsafe.class, "ensureOpen(...)");
private static final ClosedChannelException CLOSE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), AbstractUnsafe.class, "close(...)");
private static final ClosedChannelException WRITE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(null), AbstractUnsafe.class, "write(...)");
private static final ClosedChannelException FLUSH0_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(null), AbstractUnsafe.class, "flush0()");
private static final NotYetConnectedException FLUSH0_NOT_YET_CONNECTED_EXCEPTION = ThrowableUtil.unknownStackTrace(
new NotYetConnectedException(), AbstractUnsafe.class, "flush0()");
private final Channel parent;
private final ChannelId id;
private final Unsafe unsafe;
@ -596,7 +584,8 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
public final void close(final ChannelPromise promise) {
assertEventLoop();
close(promise, CLOSE_CLOSED_CHANNEL_EXCEPTION, CLOSE_CLOSED_CHANNEL_EXCEPTION, false);
ClosedChannelException closedChannelException = new ClosedChannelException();
close(promise, closedChannelException, closedChannelException, false);
}
/**
@ -621,7 +610,7 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
final ChannelOutboundBuffer outboundBuffer = this.outboundBuffer;
if (outboundBuffer == null) {
promise.setFailure(CLOSE_CLOSED_CHANNEL_EXCEPTION);
promise.setFailure(new ClosedChannelException());
return;
}
this.outboundBuffer = null; // Disallow adding any messages and flushes to outboundBuffer.
@ -821,7 +810,7 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
// need to fail the future right away. If it is not null the handling of the rest
// will be done in flush0()
// See https://github.com/netty/netty/issues/2362
safeSetFailure(promise, newWriteException(initialCloseCause));
safeSetFailure(promise, newClosedChannelException(initialCloseCause));
// release message now to prevent resource-leak
ReferenceCountUtil.release(msg);
return;
@ -877,10 +866,10 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
if (!isActive()) {
try {
if (isOpen()) {
outboundBuffer.failFlushed(FLUSH0_NOT_YET_CONNECTED_EXCEPTION, true);
outboundBuffer.failFlushed(new NotYetConnectedException(), true);
} else {
// Do not trigger channelWritabilityChanged because the channel is closed already.
outboundBuffer.failFlushed(newFlush0Exception(initialCloseCause), false);
outboundBuffer.failFlushed(newClosedChannelException(initialCloseCause), false);
}
} finally {
inFlush0 = false;
@ -901,13 +890,13 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
* may still return {@code true} even if the channel should be closed as result of the exception.
*/
initialCloseCause = t;
close(voidPromise(), t, newFlush0Exception(t), false);
close(voidPromise(), t, newClosedChannelException(t), false);
} else {
try {
shutdownOutput(voidPromise(), t);
} catch (Throwable t2) {
initialCloseCause = t;
close(voidPromise(), t2, newFlush0Exception(t), false);
close(voidPromise(), t2, newClosedChannelException(t), false);
}
}
} finally {
@ -915,28 +904,12 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
}
}
private ClosedChannelException newWriteException(Throwable cause) {
if (cause == null) {
return WRITE_CLOSED_CHANNEL_EXCEPTION;
private ClosedChannelException newClosedChannelException(Throwable cause) {
ClosedChannelException exception = new ClosedChannelException();
if (cause != null) {
exception.initCause(cause);
}
return ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(cause), AbstractUnsafe.class, "write(...)");
}
private ClosedChannelException newFlush0Exception(Throwable cause) {
if (cause == null) {
return FLUSH0_CLOSED_CHANNEL_EXCEPTION;
}
return ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(cause), AbstractUnsafe.class, "flush0()");
}
private ClosedChannelException newEnsureOpenException(Throwable cause) {
if (cause == null) {
return ENSURE_OPEN_CLOSED_CHANNEL_EXCEPTION;
}
return ThrowableUtil.unknownStackTrace(
new ExtendedClosedChannelException(cause), AbstractUnsafe.class, "ensureOpen(...)");
return exception;
}
@Override
@ -951,7 +924,7 @@ public abstract class AbstractChannel extends DefaultAttributeMap implements Cha
return true;
}
safeSetFailure(promise, newEnsureOpenException(initialCloseCause));
safeSetFailure(promise, newClosedChannelException(initialCloseCause));
return false;
}

View File

@ -15,6 +15,8 @@
*/
package io.netty.channel;
import io.netty.util.internal.UnstableApi;
/**
* A {@link RuntimeException} which is thrown when an I/O operation fails.
*/
@ -48,4 +50,10 @@ public class ChannelException extends RuntimeException {
public ChannelException(Throwable cause) {
super(cause);
}
@UnstableApi
protected ChannelException(String message, Throwable cause, boolean shared) {
super(message, cause, false, true);
assert shared;
}
}

View File

@ -30,7 +30,6 @@ import io.netty.util.ReferenceCountUtil;
import io.netty.util.concurrent.Future;
import io.netty.util.internal.InternalThreadLocalMap;
import io.netty.util.internal.PlatformDependent;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -53,10 +52,6 @@ public class LocalChannel extends AbstractChannel {
AtomicReferenceFieldUpdater.newUpdater(LocalChannel.class, Future.class, "finishReadFuture");
private static final ChannelMetadata METADATA = new ChannelMetadata(false);
private static final int MAX_READER_STACK_DEPTH = 8;
private static final ClosedChannelException DO_WRITE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), LocalChannel.class, "doWrite(...)");
private static final ClosedChannelException DO_CLOSE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), LocalChannel.class, "doClose()");
private enum State { OPEN, BOUND, CONNECTED, CLOSED }
@ -181,7 +176,7 @@ public class LocalChannel extends AbstractChannel {
ChannelPromise promise = connectPromise;
if (promise != null) {
// Use tryFailure() instead of setFailure() to avoid the race against cancel().
promise.tryFailure(DO_CLOSE_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
connectPromise = null;
}
}
@ -229,7 +224,7 @@ public class LocalChannel extends AbstractChannel {
ChannelPromise promise = connectPromise;
if (promise != null) {
// Use tryFailure() instead of setFailure() to avoid the race against cancel().
promise.tryFailure(DO_CLOSE_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
connectPromise = null;
}
}
@ -291,7 +286,7 @@ public class LocalChannel extends AbstractChannel {
case BOUND:
throw new NotYetConnectedException();
case CLOSED:
throw DO_WRITE_CLOSED_CHANNEL_EXCEPTION;
throw new ClosedChannelException();
case CONNECTED:
break;
}
@ -300,6 +295,7 @@ public class LocalChannel extends AbstractChannel {
writeInProgress = true;
try {
ClosedChannelException exception = null;
for (;;) {
Object msg = in.current();
if (msg == null) {
@ -312,7 +308,10 @@ public class LocalChannel extends AbstractChannel {
peer.inboundBuffer.add(ReferenceCountUtil.retain(msg));
in.remove();
} else {
in.remove(DO_WRITE_CLOSED_CHANNEL_EXCEPTION);
if (exception == null) {
exception = new ClosedChannelException();
}
in.remove(exception);
}
} catch (Throwable cause) {
in.remove(cause);

View File

@ -28,7 +28,6 @@ import io.netty.channel.ConnectTimeoutException;
import io.netty.channel.EventLoop;
import io.netty.util.ReferenceCountUtil;
import io.netty.util.ReferenceCounted;
import io.netty.util.internal.ThrowableUtil;
import io.netty.util.internal.logging.InternalLogger;
import io.netty.util.internal.logging.InternalLoggerFactory;
@ -49,9 +48,6 @@ public abstract class AbstractNioChannel extends AbstractChannel {
private static final InternalLogger logger =
InternalLoggerFactory.getInstance(AbstractNioChannel.class);
private static final ClosedChannelException DO_CLOSE_CLOSED_CHANNEL_EXCEPTION = ThrowableUtil.unknownStackTrace(
new ClosedChannelException(), AbstractNioChannel.class, "doClose()");
private final SelectableChannel ch;
protected final int readInterestOp;
volatile SelectionKey selectionKey;
@ -462,7 +458,7 @@ public abstract class AbstractNioChannel extends AbstractChannel {
ChannelPromise promise = connectPromise;
if (promise != null) {
// Use tryFailure() instead of setFailure() to avoid the race against cancel().
promise.tryFailure(DO_CLOSE_CLOSED_CHANNEL_EXCEPTION);
promise.tryFailure(new ClosedChannelException());
connectPromise = null;
}