You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by uc...@apache.org on 2018/05/28 09:21:29 UTC

[2/3] flink git commit: [FLINK-9386] Embed netty router

http://git-wip-us.apache.org/repos/asf/flink/blob/a5fa0931/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/router/RouterHandler.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/router/RouterHandler.java b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/router/RouterHandler.java
new file mode 100644
index 0000000..2e8200e
--- /dev/null
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/router/RouterHandler.java
@@ -0,0 +1,116 @@
+/*
+ * 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.rest.handler.router;
+
+import org.apache.flink.runtime.rest.handler.AbstractRestHandler;
+import org.apache.flink.runtime.rest.handler.util.HandlerUtils;
+import org.apache.flink.runtime.rest.messages.ErrorResponseBody;
+
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelHandler;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelHandlerContext;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelInboundHandler;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelPipeline;
+import org.apache.flink.shaded.netty4.io.netty.channel.SimpleChannelInboundHandler;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.DefaultFullHttpResponse;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpHeaders;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpMethod;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpRequest;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpResponseStatus;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpVersion;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.QueryStringDecoder;
+
+import java.util.Map;
+
+import static java.util.Objects.requireNonNull;
+
+/**
+ * Inbound handler that converts HttpRequest to Routed and passes Routed to the matched handler.
+ *
+ * <p>This class replaces the standard error response to be identical with those sent by the {@link AbstractRestHandler}.
+ *
+ * <p>This class is based on:
+ * https://github.com/sinetja/netty-router/blob/1.10/src/main/java/io/netty/handler/codec/http/router/AbstractHandler.java
+ * https://github.com/sinetja/netty-router/blob/1.10/src/main/java/io/netty/handler/codec/http/router/Handler.java
+ */
+public class RouterHandler extends SimpleChannelInboundHandler<HttpRequest> {
+	private static final String ROUTER_HANDLER_NAME = RouterHandler.class.getName() + "_ROUTER_HANDLER";
+	private static final String ROUTED_HANDLER_NAME = RouterHandler.class.getName() + "_ROUTED_HANDLER";
+
+	private final Map<String, String> responseHeaders;
+	private final Router router;
+
+	public RouterHandler(Router router, final Map<String, String> responseHeaders) {
+		this.router = requireNonNull(router);
+		this.responseHeaders = requireNonNull(responseHeaders);
+	}
+
+	public String getName() {
+		return ROUTER_HANDLER_NAME;
+	}
+
+	@Override
+	protected void channelRead0(ChannelHandlerContext channelHandlerContext, HttpRequest httpRequest) {
+		if (HttpHeaders.is100ContinueExpected(httpRequest)) {
+			channelHandlerContext.writeAndFlush(new DefaultFullHttpResponse(HttpVersion.HTTP_1_1, HttpResponseStatus.CONTINUE));
+			return;
+		}
+
+		// Route
+		HttpMethod method = httpRequest.getMethod();
+		QueryStringDecoder qsd = new QueryStringDecoder(httpRequest.getUri());
+		RouteResult<?> routeResult = router.route(method, qsd.path(), qsd.parameters());
+
+		if (routeResult == null) {
+			respondNotFound(channelHandlerContext, httpRequest);
+			return;
+		}
+
+		routed(channelHandlerContext, routeResult, httpRequest);
+	}
+
+	private void routed(
+			ChannelHandlerContext channelHandlerContext,
+			RouteResult<?> routeResult,
+			HttpRequest httpRequest) {
+		ChannelInboundHandler handler = (ChannelInboundHandler) routeResult.target();
+
+		// The handler may have been added (keep alive)
+		ChannelPipeline pipeline     = channelHandlerContext.pipeline();
+		ChannelHandler addedHandler = pipeline.get(ROUTED_HANDLER_NAME);
+		if (handler != addedHandler) {
+			if (addedHandler == null) {
+				pipeline.addAfter(ROUTER_HANDLER_NAME, ROUTED_HANDLER_NAME, handler);
+			} else {
+				pipeline.replace(addedHandler, ROUTED_HANDLER_NAME, handler);
+			}
+		}
+
+		RoutedRequest<?> request = new RoutedRequest<>(routeResult, httpRequest);
+		channelHandlerContext.fireChannelRead(request.retain());
+	}
+
+	private void respondNotFound(ChannelHandlerContext channelHandlerContext, HttpRequest request) {
+		HandlerUtils.sendErrorResponse(
+			channelHandlerContext,
+			request,
+			new ErrorResponseBody("Not found."),
+			HttpResponseStatus.NOT_FOUND,
+			responseHeaders);
+	}
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/a5fa0931/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/KeepAliveWrite.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/KeepAliveWrite.java b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/KeepAliveWrite.java
new file mode 100644
index 0000000..906613c
--- /dev/null
+++ b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/util/KeepAliveWrite.java
@@ -0,0 +1,50 @@
+/*
+ * 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.rest.handler.util;
+
+import org.apache.flink.shaded.netty4.io.netty.channel.Channel;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelFuture;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelFutureListener;
+import org.apache.flink.shaded.netty4.io.netty.channel.ChannelHandlerContext;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpHeaders;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpRequest;
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpResponse;
+
+/**
+ * Utilities to write.
+ */
+public class KeepAliveWrite {
+	public static ChannelFuture flush(ChannelHandlerContext ctx, HttpRequest request, HttpResponse response) {
+		if (!HttpHeaders.isKeepAlive(request)) {
+			return ctx.writeAndFlush(response).addListener(ChannelFutureListener.CLOSE);
+		} else {
+			response.headers().set(HttpHeaders.Names.CONNECTION, HttpHeaders.Values.KEEP_ALIVE);
+			return ctx.writeAndFlush(response);
+		}
+	}
+
+	public static ChannelFuture flush(Channel ch, HttpRequest req, HttpResponse res) {
+		if (!HttpHeaders.isKeepAlive(req)) {
+			return ch.writeAndFlush(res).addListener(ChannelFutureListener.CLOSE);
+		} else {
+			res.headers().set(HttpHeaders.Names.CONNECTION, HttpHeaders.Values.KEEP_ALIVE);
+			return ch.writeAndFlush(res);
+		}
+	}
+}

http://git-wip-us.apache.org/repos/asf/flink/blob/a5fa0931/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/TaskManagerLogHandlerTest.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/TaskManagerLogHandlerTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/TaskManagerLogHandlerTest.java
index b600cbe..2b0291f 100644
--- a/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/TaskManagerLogHandlerTest.java
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/legacy/TaskManagerLogHandlerTest.java
@@ -28,6 +28,8 @@ import org.apache.flink.runtime.instance.Instance;
 import org.apache.flink.runtime.instance.InstanceID;
 import org.apache.flink.runtime.jobmanager.slots.TaskManagerGateway;
 import org.apache.flink.runtime.jobmaster.JobManagerGateway;
+import org.apache.flink.runtime.rest.handler.router.RouteResult;
+import org.apache.flink.runtime.rest.handler.router.RoutedRequest;
 import org.apache.flink.runtime.testingUtils.TestingUtils;
 import org.apache.flink.runtime.webmonitor.retriever.GatewayRetriever;
 
@@ -36,7 +38,6 @@ import org.apache.flink.shaded.netty4.io.netty.channel.ChannelHandlerContext;
 import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.DefaultFullHttpRequest;
 import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpMethod;
 import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpVersion;
-import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.router.Routed;
 
 import org.junit.Assert;
 import org.junit.Test;
@@ -134,11 +135,16 @@ public class TaskManagerLogHandlerTest {
 
 		Map<String, String> pathParams = new HashMap<>();
 		pathParams.put(TaskManagersHandler.TASK_MANAGER_ID_KEY, tmID.toString());
-		Routed routed = mock(Routed.class);
-		when(routed.pathParams()).thenReturn(pathParams);
-		when(routed.request()).thenReturn(new DefaultFullHttpRequest(HttpVersion.HTTP_1_1, HttpMethod.GET, "/taskmanagers/" + tmID + "/log"));
-
-		handler.respondAsLeader(ctx, routed, jobManagerGateway);
+		RoutedRequest routedRequest = new RoutedRequest(
+			new RouteResult(
+				"shouldn't be used",
+				"shouldn't be used either",
+				pathParams,
+				new HashMap<>(),
+				new Object()),
+			new DefaultFullHttpRequest(HttpVersion.HTTP_1_1, HttpMethod.GET, "/taskmanagers/" + tmID + "/log"));
+
+		handler.respondAsLeader(ctx, routedRequest, jobManagerGateway);
 
 		Assert.assertEquals("Fetching TaskManager log failed.", exception.get());
 	}

http://git-wip-us.apache.org/repos/asf/flink/blob/a5fa0931/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/router/RouterTest.java
----------------------------------------------------------------------
diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/router/RouterTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/router/RouterTest.java
new file mode 100644
index 0000000..b67bc77
--- /dev/null
+++ b/flink-runtime/src/test/java/org/apache/flink/runtime/rest/handler/router/RouterTest.java
@@ -0,0 +1,180 @@
+/*
+ * 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.rest.handler.router;
+
+import org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpMethod;
+
+import org.junit.Before;
+import org.junit.Test;
+
+import java.util.Set;
+
+import static org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpMethod.GET;
+import static org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpMethod.POST;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertTrue;
+
+/**
+ * Tests for {@link Router}.
+ */
+public class RouterTest {
+	private Router<String> router;
+
+	@Before
+	public void setUp() {
+		router = StringRouter.create();
+	}
+
+	@Test
+	public void testIgnoreSlashesAtBothEnds() {
+		assertEquals("index", router.route(GET, "articles").target());
+		assertEquals("index", router.route(GET, "/articles").target());
+		assertEquals("index", router.route(GET, "//articles").target());
+		assertEquals("index", router.route(GET, "articles/").target());
+		assertEquals("index", router.route(GET, "articles//").target());
+		assertEquals("index", router.route(GET, "/articles/").target());
+		assertEquals("index", router.route(GET, "//articles//").target());
+	}
+
+	@Test
+	public void testEmptyParams() {
+		RouteResult<String> routed = router.route(GET, "/articles");
+		assertEquals("index", routed.target());
+		assertEquals(0,       routed.pathParams().size());
+	}
+
+	@Test
+	public void testParams() {
+		RouteResult<String> routed = router.route(GET, "/articles/123");
+		assertEquals("show", routed.target());
+		assertEquals(1,      routed.pathParams().size());
+		assertEquals("123",  routed.pathParams().get("id"));
+	}
+
+	@Test
+	public void testNone() {
+		RouteResult<String> routed = router.route(GET, "/noexist");
+		assertEquals("404", routed.target());
+	}
+
+	@Test
+	public void testSplatWildcard() {
+		RouteResult<String> routed = router.route(GET, "/download/foo/bar.png");
+		assertEquals("download",    routed.target());
+		assertEquals(1,             routed.pathParams().size());
+		assertEquals("foo/bar.png", routed.pathParams().get("*"));
+	}
+
+	@Test
+	public void testOrder() {
+		RouteResult<String> routed1 = router.route(GET, "/articles/new");
+		assertEquals("new", routed1.target());
+		assertEquals(0,     routed1.pathParams().size());
+
+		RouteResult<String> routed2 = router.route(GET, "/articles/123");
+		assertEquals("show", routed2.target());
+		assertEquals(1,      routed2.pathParams().size());
+		assertEquals("123",  routed2.pathParams().get("id"));
+
+		RouteResult<String> routed3 = router.route(GET, "/notfound");
+		assertEquals("404", routed3.target());
+		assertEquals(0,     routed3.pathParams().size());
+
+		RouteResult<String> routed4 = router.route(GET, "/articles/overview");
+		assertEquals("overview", routed4.target());
+		assertEquals(0,     routed4.pathParams().size());
+
+		RouteResult<String> routed5 = router.route(GET, "/articles/overview/detailed");
+		assertEquals("detailed", routed5.target());
+		assertEquals(0,     routed5.pathParams().size());
+	}
+
+	@Test
+	public void testAnyMethod() {
+		RouteResult<String> routed1 = router.route(GET, "/anyMethod");
+		assertEquals("anyMethod", routed1.target());
+		assertEquals(0,           routed1.pathParams().size());
+
+		RouteResult<String> routed2 = router.route(POST, "/anyMethod");
+		assertEquals("anyMethod", routed2.target());
+		assertEquals(0,           routed2.pathParams().size());
+	}
+
+	@Test
+	public void testRemoveByPathPattern() {
+		router.removePathPattern("/articles");
+		RouteResult<String> routed = router.route(GET, "/articles");
+		assertEquals("404", routed.target());
+	}
+
+	@Test
+	public void testAllowedMethods() {
+		assertEquals(9, router.allAllowedMethods().size());
+
+		Set<HttpMethod> methods = router.allowedMethods("/articles");
+		assertEquals(2, methods.size());
+		assertTrue(methods.contains(GET));
+		assertTrue(methods.contains(POST));
+	}
+
+	@Test
+	public void testSubclasses() {
+		Router<Class<? extends Action>> router = new Router<Class<? extends Action>>()
+			.addRoute(GET, "/articles",     Index.class)
+			.addRoute(GET, "/articles/:id", Show.class);
+
+		RouteResult<Class<? extends Action>> routed1 = router.route(GET, "/articles");
+		RouteResult<Class<? extends Action>> routed2 = router.route(GET, "/articles/123");
+		assertNotNull(routed1);
+		assertNotNull(routed2);
+		assertEquals(Index.class, routed1.target());
+		assertEquals(Show.class,  routed2.target());
+	}
+
+	private static final class StringRouter {
+		// Utility classes should not have a public or default constructor.
+		private StringRouter() { }
+
+		static Router<String> create() {
+			return new Router<String>()
+				.addGet("/articles", "index")
+				.addGet("/articles/new", "new")
+				.addGet("/articles/overview", "overview")
+				.addGet("/articles/overview/detailed", "detailed")
+				.addGet("/articles/:id", "show")
+				.addGet("/articles/:id/:format", "show")
+				.addPost("/articles", "post")
+				.addPatch("/articles/:id", "patch")
+				.addDelete("/articles/:id", "delete")
+				.addAny("/anyMethod", "anyMethod")
+				.addGet("/download/:*", "download")
+				.notFound("404");
+		}
+	}
+
+	private interface Action {
+	}
+
+	private class Index implements Action {
+	}
+
+	private class Show  implements Action {
+	}
+}