flink-issues mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From "ASF GitHub Bot (JIRA)" <j...@apache.org>
Subject [jira] [Commented] (FLINK-10075) HTTP connections to a secured REST endpoint flood the log
Date Wed, 10 Oct 2018 11:19:00 GMT

    [ https://issues.apache.org/jira/browse/FLINK-10075?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16644830#comment-16644830
] 

ASF GitHub Bot commented on FLINK-10075:
----------------------------------------

zentol closed pull request #6796: [FLINK-10075] Redirect non-ssl requests to https url if
ssl is enabled
URL: https://github.com/apache/flink/pull/6796
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/net/RedirectingSslHandler.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/net/RedirectingSslHandler.java
new file mode 100644
index 00000000000..e608d90a12d
--- /dev/null
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/net/RedirectingSslHandler.java
@@ -0,0 +1,105 @@
+/*
+ * 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.flink.runtime.net;
+
+import org.apache.flink.runtime.rest.handler.util.HandlerRedirectUtils;
+import org.apache.flink.runtime.rest.handler.util.KeepAliveWrite;
+
+import org.apache.flink.shaded.netty4.io.netty.buffer.ByteBuf;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelHandlerContext;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelInboundHandlerAdapter;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.ByteToMessageDecoder;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpRequest;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpResponse;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpResponseStatus;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpServerCodec;
+import org.apache.flink.shaded.netty4.io.netty.handler.ssl.SslHandler;
+import org.apache.flink.shaded.netty4.io.netty.util.ReferenceCountUtil;
+
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import javax.annotation.Nonnull;
+
+import java.net.InetSocketAddress;
+import java.util.List;
+import java.util.concurrent.CompletableFuture;
+
+/** SSL handler which automatically redirects Non-SSL requests to SSL address. */
+public class RedirectingSslHandler extends ByteToMessageDecoder {
+	private static final Logger log = LoggerFactory.getLogger(RedirectingSslHandler.class);
+
+	private static final String SSL_HANDLER_NAME = "ssl";
+	private static final String HTTP_CODEC_HANDLER_NAME = "http-codec";
+	private static final String NON_SSL_HANDLER_NAME = "redirecting-non-ssl";
+
+	/** the length of the ssl record header (in bytes). */
+	private static final int SSL_RECORD_HEADER_LENGTH = 5;
+
+	@Nonnull private final String confRedirectBaseUrl;
+	@Nonnull private final CompletableFuture<String> redirectBaseUrl;
+	@Nonnull private final SSLEngineFactory sslEngineFactory;
+
+	public RedirectingSslHandler(
+		@Nonnull String confRedirectHost,
+		@Nonnull CompletableFuture<String> redirectBaseUrl,
+		@Nonnull SSLEngineFactory sslEngineFactory) {
+		this.confRedirectBaseUrl = "https://" + confRedirectHost + ":";
+		this.redirectBaseUrl = redirectBaseUrl;
+		this.sslEngineFactory = sslEngineFactory;
+	}
+
+	@Override
+	protected void decode(ChannelHandlerContext context, ByteBuf in, List<Object> out)
{
+		if (in.readableBytes() >= SSL_RECORD_HEADER_LENGTH && SslHandler.isEncrypted(in))
{
+			handleSsl(context);
+		} else {
+			context.pipeline().replace(this, HTTP_CODEC_HANDLER_NAME, new HttpServerCodec());
+			context.pipeline().addAfter(HTTP_CODEC_HANDLER_NAME, NON_SSL_HANDLER_NAME, new NonSslHandler());
+		}
+	}
+
+	private void handleSsl(ChannelHandlerContext context) {
+		SslHandler sslHandler = new SslHandler(sslEngineFactory.createSSLEngine());
+		try {
+			context.pipeline().replace(this, SSL_HANDLER_NAME, sslHandler);
+		} catch (Throwable t){
+			ReferenceCountUtil.safeRelease(sslHandler.engine());
+			throw t;
+		}
+	}
+
+	private class NonSslHandler extends ChannelInboundHandlerAdapter {
+		@Override
+		public void channelRead(ChannelHandlerContext ctx, Object msg) throws Exception {
+			HttpRequest request = msg instanceof HttpRequest ? (HttpRequest) msg : null;
+			String path = request == null ? "" : request.uri();
+			String redirectAddress = getRedirectAddress(ctx);
+			log.trace("Received non-SSL request, redirecting to {}{}", redirectAddress, path);
+			HttpResponse response = HandlerRedirectUtils.getRedirectResponse(
+				redirectAddress, path, HttpResponseStatus.MOVED_PERMANENTLY);
+			KeepAliveWrite.flush(ctx, request, response);
+		}
+
+		private String getRedirectAddress(ChannelHandlerContext ctx) throws Exception {
+			return redirectBaseUrl.isDone() ? redirectBaseUrl.get() :
+				confRedirectBaseUrl + ((InetSocketAddress) (ctx.channel()).localAddress()).getPort();
+		}
+	}
+}
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/RestServerEndpoint.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/RestServerEndpoint.java
index 38da82cb7fb..43636dd8f9e 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/RestServerEndpoint.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/RestServerEndpoint.java
@@ -22,6 +22,7 @@
 import org.apache.flink.api.common.time.Time;
 import org.apache.flink.api.java.tuple.Tuple2;
 import org.apache.flink.runtime.concurrent.FutureUtils;
+import org.apache.flink.runtime.net.RedirectingSslHandler;
 import org.apache.flink.runtime.net.SSLEngineFactory;
 import org.apache.flink.runtime.rest.handler.PipelineErrorHandler;
 import org.apache.flink.runtime.rest.handler.RestHandlerSpecification;
@@ -43,7 +44,6 @@
 import org.apache.flink.shaded.netty4.io.netty.channel.socket.SocketChannel;
 import org.apache.flink.shaded.netty4.io.netty.channel.socket.nio.NioServerSocketChannel;
 import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpServerCodec;
-import org.apache.flink.shaded.netty4.io.netty.handler.ssl.SslHandler;
 import org.apache.flink.shaded.netty4.io.netty.handler.stream.ChunkedWriteHandler;
 
 import org.slf4j.Logger;
@@ -156,7 +156,8 @@ protected void initChannel(SocketChannel ch) {
 
 					// SSL should be the first handler in the pipeline
 					if (sslEngineFactory != null) {
-						ch.pipeline().addLast("ssl", new SslHandler(sslEngineFactory.createSSLEngine()));
+						ch.pipeline().addLast("ssl",
+							new RedirectingSslHandler(restAddress, restAddressFuture, sslEngineFactory));
 					}
 
 					ch.pipeline()
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/HandlerRedirectUtils.java
b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/HandlerRedirectUtils.java
index 900e1d24da1..eed1c71c736 100644
--- a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/HandlerRedirectUtils.java
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/HandlerRedirectUtils.java
@@ -68,13 +68,16 @@
 	}
 
 	public static HttpResponse getRedirectResponse(String redirectAddress, String path) {
+		return getRedirectResponse(redirectAddress, path, HttpResponseStatus.TEMPORARY_REDIRECT);
+	}
+
+	public static HttpResponse getRedirectResponse(String redirectAddress, String path, HttpResponseStatus
code) {
 		checkNotNull(redirectAddress, "Redirect address");
 		checkNotNull(path, "Path");
 
 		String newLocation = String.format("%s%s", redirectAddress, path);
 
-		HttpResponse redirectResponse = new DefaultFullHttpResponse(
-				HttpVersion.HTTP_1_1, HttpResponseStatus.TEMPORARY_REDIRECT);
+		HttpResponse redirectResponse = new DefaultFullHttpResponse(HttpVersion.HTTP_1_1, code);
 		redirectResponse.headers().set(HttpHeaders.Names.LOCATION, newLocation);
 		redirectResponse.headers().set(HttpHeaders.Names.CONTENT_LENGTH, 0);
 
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/rest/RestServerEndpointITCase.java
b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/RestServerEndpointITCase.java
index 962619160e4..430bfad38fd 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/rest/RestServerEndpointITCase.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/RestServerEndpointITCase.java
@@ -96,6 +96,7 @@
 import java.util.stream.Collectors;
 
 import static java.util.Objects.requireNonNull;
+import static org.hamcrest.CoreMatchers.hasItems;
 import static org.hamcrest.Matchers.containsString;
 import static org.hamcrest.Matchers.instanceOf;
 import static org.junit.Assert.assertEquals;
@@ -530,6 +531,20 @@ public void testDefaultVersionRouting() throws Exception {
 		}
 	}
 
+	@Test
+	public void testNonSslRedirectForEnabledSsl() throws Exception {
+		Assume.assumeTrue(config.getBoolean(SecurityOptions.SSL_REST_ENABLED));
+		OkHttpClient client = new OkHttpClient.Builder().followRedirects(false).build();
+		String httpsUrl = serverEndpoint.getRestBaseUrl() + "/path";
+		String httpUrl = httpsUrl.replace("https://", "http://");
+		Request request = new Request.Builder().url(httpUrl).build();
+		try (final Response response = client.newCall(request).execute()) {
+			assertEquals(HttpResponseStatus.MOVED_PERMANENTLY.code(), response.code());
+			assertThat(response.headers().names(), hasItems("Location"));
+			assertEquals(httpsUrl, response.header("Location"));
+		}
+	}
+
 	private HttpURLConnection openHttpConnectionForUpload(final String boundary) throws IOException
{
 		final HttpURLConnection connection =
 			(HttpURLConnection) new URL(serverEndpoint.getRestBaseUrl() + "/upload").openConnection();


 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
users@infra.apache.org


> HTTP connections to a secured REST endpoint flood the log
> ---------------------------------------------------------
>
>                 Key: FLINK-10075
>                 URL: https://issues.apache.org/jira/browse/FLINK-10075
>             Project: Flink
>          Issue Type: Improvement
>          Components: REST
>    Affects Versions: 1.5.2, 1.6.0, 1.7.0
>            Reporter: Stephan Ewen
>            Assignee: Andrey Zagrebin
>            Priority: Critical
>              Labels: pull-request-available
>             Fix For: 1.7.0, 1.6.2, 1.5.5
>
>
> When connecting with a browser (or other client tool) to a secured REST endpoint, the
decoder throws many exceptions indicating that the received data is not an SSL record.
> This massively floods the log, drowning out everything else (see below).
> h2. Proposed Solution
> If a {{NotSslRecordException}} is caught, Netty should send a response HTTP 301 with
a new location of https://host:port/
> The response would need to bypass the SSL handler because it must come in plain text.
> h2. Fallback Solution
> If the proper solution cannot work, we should reduce the log level for that particular
exception to *TRACE*.
> h2. Sample Log Output
> Log message that is written per each request (there are many per web UI page)
> {code}
> 2018-08-06 19:07:57,734 WARN  org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint
   - Unhandled exception
> org.apache.flink.shaded.netty4.io.netty.handler.codec.DecoderException: org.apache.flink.shaded.netty4.io.netty.handler.ssl.NotSslRecordException:
not an SSL/TLS record: 474554
> 202f7061727469616c732f6f766572766965772e68746d6c20485454502f312e310d0a486f73743a206c6f63616c686f73743a383038310d0a436f6e6e656374696f6e3a206b6565702d616c6976650d0a4163636570743a
> 20746578742f68746d6c0d0a557365722d4167656e743a204d6f7a696c6c612f352e3020285831313b204c696e7578207838365f363429204170706c655765624b69742f3533372e333620284b48544d4c2c206c696b6520
> 4765636b6f29204368726f6d652f34372e302e323532362e313131205361666172692f3533372e33360d0a526566657265723a20687474703a2f2f6c6f63616c686f73743a383038312f0d0a4163636570742d456e636f64
> 696e673a20677a69702c206465666c6174652c20736463680d0a4163636570742d4c616e67756167653a20656e2d55532c656e3b713d302e382c64653b713d302e360d0a49662d4d6f6469666965642d53696e63653a204d
> 6f6e2c2030362041756720323031382031353a34343a313720474d540d0a0d0a
>         at org.apache.flink.shaded.netty4.io.netty.handler.codec.ByteToMessageDecoder.callDecode(ByteToMessageDecoder.java:459)
>         at org.apache.flink.shaded.netty4.io.netty.handler.codec.ByteToMessageDecoder.channelRead(ByteToMessageDecoder.java:265)
>         at org.apache.flink.shaded.netty4.io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:362)
>         at org.apache.flink.shaded.netty4.io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:348)
>         at org.apache.flink.shaded.netty4.io.netty.channel.AbstractChannelHandlerContext.fireChannelRead(AbstractChannelHandlerContext.java:340)
>         at org.apache.flink.shaded.netty4.io.netty.channel.DefaultChannelPipeline$HeadContext.channelRead(DefaultChannelPipeline.java:1434)
>         at org.apache.flink.shaded.netty4.io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:362)
>         at org.apache.flink.shaded.netty4.io.netty.channel.AbstractChannelHandlerContext.invokeChannelRead(AbstractChannelHandlerContext.java:348)
>         at org.apache.flink.shaded.netty4.io.netty.channel.DefaultChannelPipeline.fireChannelRead(DefaultChannelPipeline.java:965)
>         at org.apache.flink.shaded.netty4.io.netty.channel.nio.AbstractNioByteChannel$NioByteUnsafe.read(AbstractNioByteChannel.java:163)
>         at org.apache.flink.shaded.netty4.io.netty.channel.nio.NioEventLoop.processSelectedKey(NioEventLoop.java:645)
>         at org.apache.flink.shaded.netty4.io.netty.channel.nio.NioEventLoop.processSelectedKeysOptimized(NioEventLoop.java:580)
>         at org.apache.flink.shaded.netty4.io.netty.channel.nio.NioEventLoop.processSelectedKeys(NioEventLoop.java:497)
>         at org.apache.flink.shaded.netty4.io.netty.channel.nio.NioEventLoop.run(NioEventLoop.java:459)
>         at org.apache.flink.shaded.netty4.io.netty.util.concurrent.SingleThreadEventExecutor$5.run(SingleThreadEventExecutor.java:884)
>         at org.apache.flink.shaded.netty4.io.netty.util.concurrent.FastThreadLocalRunnable.run(FastThreadLocalRunnable.java:30)
>         at java.lang.Thread.run(Thread.java:745)
> {code}



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Mime
View raw message