[FLINK-7806] [flip6] Register CurrentJobsOverviewHandler under /jobs/overview
Rename CurrentJobsOverviewHandler to JobsOverviewHandler Change paths Remove joboverview/running and joboverview/completed from JobsOverviewHandler Adapt web ui files Update rest_api to reflect new REST call /jobs This changes #4805. Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/8086e3be Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/8086e3be Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/8086e3be Branch: refs/heads/master Commit: 8086e3bee8be4614359041c14786140edff19666 Parents: 0c62c52 Author: Till Rohrmann <trohrm...@apache.org> Authored: Wed Oct 11 12:39:06 2017 +0200 Committer: Till Rohrmann <trohrm...@apache.org> Committed: Tue Nov 7 15:07:44 2017 +0100 ---------------------------------------------------------------------- docs/monitoring/historyserver.md | 2 +- docs/monitoring/rest_api.md | 33 +--- .../client/program/rest/RestClusterClient.java | 4 +- .../program/rest/RestClusterClientTest.java | 4 +- .../runtime/webmonitor/WebRuntimeMonitor.java | 20 +-- .../history/HistoryServerArchiveFetcher.java | 4 +- .../webmonitor/history/HistoryServerTest.java | 6 +- .../app/scripts/modules/jobs/jobs.svc.coffee | 2 +- .../web-dashboard/web/js/hs/index.js | 4 +- flink-runtime-web/web-dashboard/web/js/index.js | 4 +- .../runtime/akka/AkkaJobManagerGateway.java | 2 +- .../flink/runtime/dispatcher/Dispatcher.java | 2 +- .../dispatcher/DispatcherRestEndpoint.java | 16 +- .../rest/handler/job/JobsOverviewHandler.java | 62 +++++++ .../handler/legacy/ClusterOverviewHandler.java | 4 +- .../legacy/CurrentJobsOverviewHandler.java | 168 ------------------- .../handler/legacy/JobsOverviewHandler.java | 134 +++++++++++++++ .../handler/legacy/metrics/MetricFetcher.java | 7 +- .../rest/messages/ClusterOverviewHeaders.java | 4 +- .../CurrentJobsOverviewHandlerHeaders.java | 70 -------- .../rest/messages/JobsOverviewHeaders.java | 72 ++++++++ .../runtime/webmonitor/RestfulGateway.java | 4 - .../legacy/CurrentJobsOverviewHandlerTest.java | 119 ------------- .../handler/legacy/JobsOverviewHandlerTest.java | 111 ++++++++++++ .../legacy/metrics/MetricFetcherTest.java | 3 +- 25 files changed, 421 insertions(+), 440 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/docs/monitoring/historyserver.md ---------------------------------------------------------------------- diff --git a/docs/monitoring/historyserver.md b/docs/monitoring/historyserver.md index 61660a5..dfbbc8e 100644 --- a/docs/monitoring/historyserver.md +++ b/docs/monitoring/historyserver.md @@ -80,7 +80,7 @@ Below is a list of available requests, with a sample JSON response. All requests Values in angle brackets are variables, for example `http://hostname:port/jobs/<jobid>/exceptions` will have to requested for example as `http://hostname:port/jobs/7684be6004e4e955c2a558a9bc463f65/exceptions`. - `/config` - - `/joboverview` + - `/jobs/overview` - `/jobs/<jobid>` - `/jobs/<jobid>/vertices` - `/jobs/<jobid>/config` http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/docs/monitoring/rest_api.md ---------------------------------------------------------------------- diff --git a/docs/monitoring/rest_api.md b/docs/monitoring/rest_api.md index e74d8b7..5ac6652 100644 --- a/docs/monitoring/rest_api.md +++ b/docs/monitoring/rest_api.md @@ -55,9 +55,7 @@ Values in angle brackets are variables, for example `http://hostname:8081/jobs/< - `/config` - `/overview` - - `/jobs` - - `/joboverview/running` - - `/joboverview/completed` + - `/jobs/overview` - `/jobs/<jobid>` - `/jobs/<jobid>/vertices` - `/jobs/<jobid>/config` @@ -117,22 +115,7 @@ Sample Result: ### Overview of Jobs -**`/jobs`** - -IDs of the jobs, grouped by status *running*, *finished*, *failed*, *canceled*. - -Sample Result: - -~~~ -{ - "jobs-running": [], - "jobs-finished": ["7684be6004e4e955c2a558a9bc463f65","49306f94d0920216b636e8dd503a6409"], - "jobs-cancelled":[], - "jobs-failed":[] -} -~~~ - -**`/joboverview`** +**`/jobs/overview`** Jobs, grouped by status, each with a small summary of its status. @@ -140,8 +123,7 @@ Sample Result: ~~~ { - "running":[], - "finished":[ + "jobs":[ { "jid": "7684be6004e4e955c2a558a9bc463f65", "name": "Flink Java Job at Wed Sep 16 18:08:21 CEST 2015", @@ -168,15 +150,6 @@ Sample Result: } ~~~ -**`/joboverview/running`** - -Jobs, grouped by status, each with a small summary of its status. The same as `/joboverview`, but containing only currently running jobs. - -**`/joboverview/completed`** - -Jobs, grouped by status, each with a small summary of its status. The same as `/joboverview`, but containing only completed (finished, canceled, or failed) jobs. - - ### Details of a Running or Completed Job **`/jobs/<jobid>`** http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-clients/src/main/java/org/apache/flink/client/program/rest/RestClusterClient.java ---------------------------------------------------------------------- diff --git a/flink-clients/src/main/java/org/apache/flink/client/program/rest/RestClusterClient.java b/flink-clients/src/main/java/org/apache/flink/client/program/rest/RestClusterClient.java index 3916514..012bedc 100644 --- a/flink-clients/src/main/java/org/apache/flink/client/program/rest/RestClusterClient.java +++ b/flink-clients/src/main/java/org/apache/flink/client/program/rest/RestClusterClient.java @@ -35,10 +35,10 @@ import org.apache.flink.runtime.messages.webmonitor.MultipleJobsDetails; import org.apache.flink.runtime.rest.RestClient; import org.apache.flink.runtime.rest.messages.BlobServerPortHeaders; import org.apache.flink.runtime.rest.messages.BlobServerPortResponseBody; -import org.apache.flink.runtime.rest.messages.CurrentJobsOverviewHandlerHeaders; import org.apache.flink.runtime.rest.messages.EmptyResponseBody; import org.apache.flink.runtime.rest.messages.JobTerminationHeaders; import org.apache.flink.runtime.rest.messages.JobTerminationMessageParameters; +import org.apache.flink.runtime.rest.messages.JobsOverviewHeaders; import org.apache.flink.runtime.rest.messages.TerminationModeQueryParameter; import org.apache.flink.runtime.rest.messages.job.JobSubmitHeaders; import org.apache.flink.runtime.rest.messages.job.JobSubmitRequestBody; @@ -199,7 +199,7 @@ public class RestClusterClient extends ClusterClient { @Override public CompletableFuture<Collection<JobStatusMessage>> listJobs() throws Exception { - CurrentJobsOverviewHandlerHeaders headers = CurrentJobsOverviewHandlerHeaders.getInstance(); + JobsOverviewHeaders headers = JobsOverviewHeaders.getInstance(); CompletableFuture<MultipleJobsDetails> jobDetailsFuture = restClient.sendRequest( restClusterClientConfiguration.getRestServerAddress(), restClusterClientConfiguration.getRestServerPort(), http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-clients/src/test/java/org/apache/flink/client/program/rest/RestClusterClientTest.java ---------------------------------------------------------------------- diff --git a/flink-clients/src/test/java/org/apache/flink/client/program/rest/RestClusterClientTest.java b/flink-clients/src/test/java/org/apache/flink/client/program/rest/RestClusterClientTest.java index 3b6ddaf..7e8185c 100644 --- a/flink-clients/src/test/java/org/apache/flink/client/program/rest/RestClusterClientTest.java +++ b/flink-clients/src/test/java/org/apache/flink/client/program/rest/RestClusterClientTest.java @@ -38,12 +38,12 @@ import org.apache.flink.runtime.rest.handler.RestHandlerException; import org.apache.flink.runtime.rest.handler.RestHandlerSpecification; import org.apache.flink.runtime.rest.messages.BlobServerPortHeaders; import org.apache.flink.runtime.rest.messages.BlobServerPortResponseBody; -import org.apache.flink.runtime.rest.messages.CurrentJobsOverviewHandlerHeaders; import org.apache.flink.runtime.rest.messages.EmptyMessageParameters; import org.apache.flink.runtime.rest.messages.EmptyRequestBody; import org.apache.flink.runtime.rest.messages.EmptyResponseBody; import org.apache.flink.runtime.rest.messages.JobTerminationHeaders; import org.apache.flink.runtime.rest.messages.JobTerminationMessageParameters; +import org.apache.flink.runtime.rest.messages.JobsOverviewHeaders; import org.apache.flink.runtime.rest.messages.MessageHeaders; import org.apache.flink.runtime.rest.messages.MessageParameters; import org.apache.flink.runtime.rest.messages.RequestBody; @@ -307,7 +307,7 @@ public class RestClusterClientTest extends TestLogger { private static class TestListJobsHandler extends TestHandler<EmptyRequestBody, MultipleJobsDetails, EmptyMessageParameters> { private TestListJobsHandler() { - super(CurrentJobsOverviewHandlerHeaders.getInstance()); + super(JobsOverviewHeaders.getInstance()); } @Override http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/WebRuntimeMonitor.java ---------------------------------------------------------------------- diff --git a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/WebRuntimeMonitor.java b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/WebRuntimeMonitor.java index fe5f106..445c61c 100644 --- a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/WebRuntimeMonitor.java +++ b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/WebRuntimeMonitor.java @@ -33,7 +33,6 @@ import org.apache.flink.runtime.rest.handler.legacy.ClusterConfigHandler; import org.apache.flink.runtime.rest.handler.legacy.ClusterOverviewHandler; import org.apache.flink.runtime.rest.handler.legacy.ConstantTextHandler; import org.apache.flink.runtime.rest.handler.legacy.CurrentJobIdsHandler; -import org.apache.flink.runtime.rest.handler.legacy.CurrentJobsOverviewHandler; import org.apache.flink.runtime.rest.handler.legacy.DashboardConfigHandler; import org.apache.flink.runtime.rest.handler.legacy.ExecutionGraphCache; import org.apache.flink.runtime.rest.handler.legacy.JobAccumulatorsHandler; @@ -48,6 +47,7 @@ import org.apache.flink.runtime.rest.handler.legacy.JobVertexAccumulatorsHandler import org.apache.flink.runtime.rest.handler.legacy.JobVertexBackPressureHandler; import org.apache.flink.runtime.rest.handler.legacy.JobVertexDetailsHandler; import org.apache.flink.runtime.rest.handler.legacy.JobVertexTaskManagersHandler; +import org.apache.flink.runtime.rest.handler.legacy.JobsOverviewHandler; import org.apache.flink.runtime.rest.handler.legacy.RequestHandler; import org.apache.flink.runtime.rest.handler.legacy.SubtaskCurrentAttemptDetailsHandler; import org.apache.flink.runtime.rest.handler.legacy.SubtaskExecutionAttemptAccumulatorsHandler; @@ -271,6 +271,13 @@ public class WebRuntimeMonitor implements WebMonitor { get(router, new AggregatingTaskManagersMetricsHandler(scheduledExecutor, metricFetcher)); get(router, new TaskManagerMetricsHandler(scheduledExecutor, metricFetcher)); + // overview over jobs + get(router, new JobsOverviewHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT)); + + get(router, new CurrentJobIdsHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT)); + + get(router, new JobDetailsHandler(executionGraphCache, scheduledExecutor, metricFetcher)); + get(router, new AggregatingJobsMetricsHandler(scheduledExecutor, metricFetcher)); get(router, new JobMetricsHandler(scheduledExecutor, metricFetcher)); @@ -279,15 +286,6 @@ public class WebRuntimeMonitor implements WebMonitor { get(router, new AggregatingSubtasksMetricsHandler(scheduledExecutor, metricFetcher)); get(router, new SubtaskMetricsHandler(scheduledExecutor, metricFetcher)); - // overview over jobs - get(router, new CurrentJobsOverviewHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT, true, true)); - get(router, new CurrentJobsOverviewHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT, true, false)); - get(router, new CurrentJobsOverviewHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT, false, true)); - - get(router, new CurrentJobIdsHandler(scheduledExecutor, DEFAULT_REQUEST_TIMEOUT)); - - get(router, new JobDetailsHandler(executionGraphCache, scheduledExecutor, metricFetcher)); - get(router, new JobVertexDetailsHandler(executionGraphCache, scheduledExecutor, metricFetcher)); get(router, new SubtasksTimesHandler(executionGraphCache, scheduledExecutor)); get(router, new JobVertexTaskManagersHandler(executionGraphCache, scheduledExecutor, metricFetcher)); @@ -421,7 +419,7 @@ public class WebRuntimeMonitor implements WebMonitor { */ public static JsonArchivist[] getJsonArchivists() { JsonArchivist[] archivists = { - new CurrentJobsOverviewHandler.CurrentJobsOverviewJsonArchivist(), + new JobsOverviewHandler.CurrentJobsOverviewJsonArchivist(), new JobPlanHandler.JobPlanJsonArchivist(), new JobConfigHandler.JobConfigJsonArchivist(), http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/history/HistoryServerArchiveFetcher.java ---------------------------------------------------------------------- diff --git a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/history/HistoryServerArchiveFetcher.java b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/history/HistoryServerArchiveFetcher.java index cd0c92d..58d532b 100644 --- a/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/history/HistoryServerArchiveFetcher.java +++ b/flink-runtime-web/src/main/java/org/apache/flink/runtime/webmonitor/history/HistoryServerArchiveFetcher.java @@ -24,7 +24,7 @@ import org.apache.flink.core.fs.FileStatus; import org.apache.flink.core.fs.FileSystem; import org.apache.flink.core.fs.Path; import org.apache.flink.runtime.history.FsJobArchivist; -import org.apache.flink.runtime.rest.handler.legacy.CurrentJobsOverviewHandler; +import org.apache.flink.runtime.rest.handler.legacy.JobsOverviewHandler; import org.apache.flink.runtime.util.ExecutorThreadFactory; import org.apache.flink.util.FileUtils; @@ -224,7 +224,7 @@ class HistoryServerArchiveFetcher { } /** - * This method replicates the JSON response that would be given by the {@link CurrentJobsOverviewHandler} when + * This method replicates the JSON response that would be given by the {@link JobsOverviewHandler} when * listing both running and finished jobs. * * <p>Every job archive contains a joboverview.json file containing the same structure. Since jobs are archived on http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-runtime-web/src/test/java/org/apache/flink/runtime/webmonitor/history/HistoryServerTest.java ---------------------------------------------------------------------- diff --git a/flink-runtime-web/src/test/java/org/apache/flink/runtime/webmonitor/history/HistoryServerTest.java b/flink-runtime-web/src/test/java/org/apache/flink/runtime/webmonitor/history/HistoryServerTest.java index 003ef04..b270ca7 100644 --- a/flink-runtime-web/src/test/java/org/apache/flink/runtime/webmonitor/history/HistoryServerTest.java +++ b/flink-runtime-web/src/test/java/org/apache/flink/runtime/webmonitor/history/HistoryServerTest.java @@ -28,6 +28,8 @@ import org.apache.flink.runtime.jobmanager.JobManager; import org.apache.flink.runtime.jobmanager.MemoryArchivist; import org.apache.flink.runtime.messages.ArchiveMessages; import org.apache.flink.runtime.rest.handler.legacy.utils.ArchivedJobGenerationUtils; +import org.apache.flink.runtime.rest.messages.JobsOverviewHeaders; +import org.apache.flink.util.TestLogger; import org.apache.flink.shaded.jackson2.com.fasterxml.jackson.databind.JsonNode; import org.apache.flink.shaded.jackson2.com.fasterxml.jackson.databind.ObjectMapper; @@ -53,7 +55,7 @@ import scala.Option; /** * Tests for the HistoryServer. */ -public class HistoryServerTest { +public class HistoryServerTest extends TestLogger { @Rule public TemporaryFolder tmpDir = new TemporaryFolder(); @@ -91,7 +93,7 @@ public class HistoryServerTest { numFinishedPolls.await(10L, TimeUnit.SECONDS); ObjectMapper mapper = new ObjectMapper(); - String response = getFromHTTP(baseUrl + "/joboverview"); + String response = getFromHTTP(baseUrl + JobsOverviewHeaders.URL); JsonNode overview = mapper.readTree(response); String jobID = overview.get("finished").get(0).get("jid").asText(); http://git-wip-us.apache.org/repos/asf/flink/blob/8086e3be/flink-runtime-web/web-dashboard/app/scripts/modules/jobs/jobs.svc.coffee ---------------------------------------------------------------------- diff --git a/flink-runtime-web/web-dashboard/app/scripts/modules/jobs/jobs.svc.coffee b/flink-runtime-web/web-dashboard/app/scripts/modules/jobs/jobs.svc.coffee index 7351de8..f07d9c0 100644 --- a/flink-runtime-web/web-dashboard/app/scripts/modules/jobs/jobs.svc.coffee +++ b/flink-runtime-web/web-dashboard/app/scripts/modules/jobs/jobs.svc.coffee @@ -86,7 +86,7 @@ angular.module('flinkApp') @listJobs = -> deferred = $q.defer() - $http.get flinkConfig.jobServer + "joboverview" + $http.get flinkConfig.jobServer + "jobs/overview" .success (data, status, headers, config) => angular.forEach data, (list, listKey) => switch listKey