This is an automated email from the ASF dual-hosted git repository.

nkruber pushed a commit to branch release-1.13
in repository https://gitbox.apache.org/repos/asf/flink.git


The following commit(s) were added to refs/heads/release-1.13 by this push:
     new 39ad632  [FLINK-23102][rest] Return empty FlameGraph if feature is 
disabled
39ad632 is described below

commit 39ad632a60896c94fb796e46c1bacfea32ea68d8
Author: Yao Zhang <xzhang...@126.com>
AuthorDate: Fri Jul 2 14:58:25 2021 +0800

    [FLINK-23102][rest] Return empty FlameGraph if feature is disabled
---
 .../shortcodes/generated/rest_v1_dispatcher.html   | 100 +++++++++++++++++++++
 .../src/test/resources/rest_api_v1.snapshot        |  51 ++++++++++-
 .../job-overview-drawer-flamegraph.component.html  |  10 ++-
 .../handler/job/JobVertexFlameGraphHandler.java    |  41 ++++++++-
 .../runtime/webmonitor/WebMonitorEndpoint.java     |  16 ++--
 .../webmonitor/threadinfo/JobVertexFlameGraph.java |  13 ++-
 6 files changed, 218 insertions(+), 13 deletions(-)

diff --git a/docs/layouts/shortcodes/generated/rest_v1_dispatcher.html 
b/docs/layouts/shortcodes/generated/rest_v1_dispatcher.html
index fca2212..b68cf80 100644
--- a/docs/layouts/shortcodes/generated/rest_v1_dispatcher.html
+++ b/docs/layouts/shortcodes/generated/rest_v1_dispatcher.html
@@ -3777,6 +3777,106 @@ Using 'curl' you can upload a jar via 'curl -X POST -H 
"Expect:" -F "jarfile=@pa
 <table class="rest-api table table-bordered">
   <tbody>
     <tr>
+      <td class="text-left" 
colspan="2"><h5><strong>/jobs/:jobid/vertices/:vertexid/flamegraph</strong></h5></td>
+    </tr>
+    <tr>
+      <td class="text-left" style="width: 20%">Verb: <code>GET</code></td>
+      <td class="text-left">Response code: <code>200 OK</code></td>
+    </tr>
+    <tr>
+      <td colspan="2">Returns flame graph information for a vertex, and may 
initiate flame graph sampling if necessary.</td>
+    </tr>
+    <tr>
+      <td colspan="2">Path parameters</td>
+    </tr>
+    <tr>
+      <td colspan="2">
+        <ul>
+<li><code>jobid</code> - 32-character hexadecimal string value that identifies 
a job.</li>
+<li><code>vertexid</code> - 32-character hexadecimal string value that 
identifies a job vertex.</li>
+        </ul>
+      </td>
+    </tr>
+    <tr>
+      <td colspan="2">Query parameters</td>
+    </tr>
+    <tr>
+      <td colspan="2">
+        <ul>
+<li><code>type</code> (optional): String value that specifies the Flame Graph 
type. Supported options are: "[FULL, ON_CPU, OFF_CPU]".</li>
+        </ul>
+      </td>
+    </tr>
+    <tr>
+      <td colspan="2">
+      <div class="book-expand">
+        <label>
+          <div class="book-expand-head flex justify-between">
+            <span>Request</span>
+            &nbsp;            <span>▾</span>
+          </div>
+          <input type="checkbox" class="hidden">
+          <div class="book-expand-content markdown-inner">
+          <pre>
+            <code>
+{}            </code>
+          </pre>
+          </div>
+        </label>
+      </div>
+      </td>
+    </tr>
+    <tr>
+      <td colspan="2">
+      <div class="book-expand">
+        <label>
+          <div class="book-expand-head flex justify-between">
+            <span>Response</span>
+            &nbsp;            <span>▾</span>
+          </div>
+          <input type="checkbox" class="hidden">
+          <div class="book-expand-content markdown-inner">
+          <pre>
+            <code>
+{
+  "type" : "object",
+  "id" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph",
+  "properties" : {
+    "data" : {
+      "type" : "object",
+      "id" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph:Node",
+      "properties" : {
+        "children" : {
+          "type" : "array",
+          "items" : {
+            "type" : "object",
+            "$ref" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph:Node"
+          }
+        },
+        "name" : {
+          "type" : "string"
+        },
+        "value" : {
+          "type" : "integer"
+        }
+      }
+    },
+    "endTimestamp" : {
+      "type" : "integer"
+    }
+  }
+}            </code>
+          </pre>
+          </div>
+        </label>
+      </div>
+      </td>
+    </tr>
+  </tbody>
+</table>
+<table class="rest-api table table-bordered">
+  <tbody>
+    <tr>
       <td class="text-left" 
colspan="2"><h5><strong>/jobs/:jobid/vertices/:vertexid/metrics</strong></h5></td>
     </tr>
     <tr>
diff --git a/flink-runtime-web/src/test/resources/rest_api_v1.snapshot 
b/flink-runtime-web/src/test/resources/rest_api_v1.snapshot
index bf1f190..7040dd8 100644
--- a/flink-runtime-web/src/test/resources/rest_api_v1.snapshot
+++ b/flink-runtime-web/src/test/resources/rest_api_v1.snapshot
@@ -2131,6 +2131,55 @@
       }
     }
   }, {
+    "url" : "/jobs/:jobid/vertices/:vertexid/flamegraph",
+    "method" : "GET",
+    "status-code" : "200 OK",
+    "file-upload" : false,
+    "path-parameters" : {
+      "pathParameters" : [ {
+        "key" : "jobid"
+      }, {
+        "key" : "vertexid"
+      } ]
+    },
+    "query-parameters" : {
+      "queryParameters" : [ {
+        "key" : "type",
+        "mandatory" : false
+      } ]
+    },
+    "request" : {
+      "type" : "any"
+    },
+    "response" : {
+      "type" : "object",
+      "id" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph",
+      "properties" : {
+        "endTimestamp" : {
+          "type" : "integer"
+        },
+        "data" : {
+          "type" : "object",
+          "id" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph:Node",
+          "properties" : {
+            "name" : {
+              "type" : "string"
+            },
+            "value" : {
+              "type" : "integer"
+            },
+            "children" : {
+              "type" : "array",
+              "items" : {
+                "type" : "object",
+                "$ref" : 
"urn:jsonschema:org:apache:flink:runtime:webmonitor:threadinfo:JobVertexFlameGraph:Node"
+              }
+            }
+          }
+        }
+      }
+    }
+  }, {
     "url" : "/jobs/:jobid/vertices/:vertexid/metrics",
     "method" : "GET",
     "status-code" : "200 OK",
@@ -3274,4 +3323,4 @@
       }
     }
   } ]
-}
+}
\ No newline at end of file
diff --git 
a/flink-runtime-web/web-dashboard/src/app/pages/job/overview/flamegraph/job-overview-drawer-flamegraph.component.html
 
b/flink-runtime-web/web-dashboard/src/app/pages/job/overview/flamegraph/job-overview-drawer-flamegraph.component.html
index 1c108d6..4a55f4b 100644
--- 
a/flink-runtime-web/web-dashboard/src/app/pages/job/overview/flamegraph/job-overview-drawer-flamegraph.component.html
+++ 
b/flink-runtime-web/web-dashboard/src/app/pages/job/overview/flamegraph/job-overview-drawer-flamegraph.component.html
@@ -28,10 +28,12 @@ Type:
   <label nz-radio-button nzValue="full" (click)="selectFrameGraphType()" 
title="Includes stack traces of threads in all states">Mixed</label>
 </nz-radio-group>
 
-Measurement:
-<span *ngIf="flameGraph.endTimestamp">
-  {{ (now - flameGraph.endTimestamp) | humanizeDuration }} ago
-</span>
+<ng-container [ngSwitch]="flameGraph.endTimestamp">
+  <span *ngSwitchCase="-1">The task has already been terminated</span>
+  <span *ngSwitchCase="-2">The flame graph feature is currently disabled 
(enable it by setting rest.flamegraph.enabled: true)</span>
+  <span *ngSwitchCase="-3">We are waiting for the first samples to create a 
flame graph</span>
+  <span *ngSwitchDefault>Measurement: {{ (now - flameGraph.endTimestamp) | 
humanizeDuration }} ago</span>
+</ng-container>
 
 <div class="flame-graph-area">
   <flink-flame-graph [data]="flameGraph['data']" 
[graphType]="graphType"></flink-flame-graph>
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/job/JobVertexFlameGraphHandler.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/job/JobVertexFlameGraphHandler.java
index b936477..e082148 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/job/JobVertexFlameGraphHandler.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/rest/handler/job/JobVertexFlameGraphHandler.java
@@ -20,6 +20,7 @@ package org.apache.flink.runtime.rest.handler.job;
 
 import org.apache.flink.api.common.time.Time;
 import org.apache.flink.runtime.executiongraph.AccessExecutionJobVertex;
+import org.apache.flink.runtime.rest.handler.AbstractRestHandler;
 import org.apache.flink.runtime.rest.handler.HandlerRequest;
 import org.apache.flink.runtime.rest.handler.RestHandlerException;
 import org.apache.flink.runtime.rest.handler.legacy.ExecutionGraphCache;
@@ -37,9 +38,12 @@ import 
org.apache.flink.runtime.webmonitor.threadinfo.JobVertexThreadInfoTracker
 
 import 
org.apache.flink.shaded.netty4.io.netty.handler.codec.http.HttpResponseStatus;
 
+import javax.annotation.Nonnull;
+
 import java.util.List;
 import java.util.Map;
 import java.util.Optional;
+import java.util.concurrent.CompletableFuture;
 import java.util.concurrent.Executor;
 
 /** Request handler for the job vertex Flame Graph. */
@@ -72,7 +76,7 @@ public class JobVertexFlameGraphHandler
             throws RestHandlerException {
 
         if (jobVertex.getAggregateState().isTerminal()) {
-            return JobVertexFlameGraph.empty();
+            return JobVertexFlameGraph.terminated();
         }
 
         final Optional<JobVertexThreadInfoStats> threadInfoSample =
@@ -102,7 +106,7 @@ public class JobVertexFlameGraphHandler
                         HttpResponseStatus.BAD_REQUEST);
         }
 
-        return operatorFlameGraph.orElse(JobVertexFlameGraph.empty());
+        return operatorFlameGraph.orElse(JobVertexFlameGraph.waiting());
     }
 
     private static FlameGraphTypeQueryParameter.Type getFlameGraphType(
@@ -121,4 +125,37 @@ public class JobVertexFlameGraphHandler
     public void close() throws Exception {
         threadInfoOperatorTracker.shutDown();
     }
+
+    public static AbstractRestHandler<?, ?, ?, ?> disabledHandler(
+            GatewayRetriever<? extends RestfulGateway> leaderRetriever,
+            Time timeout,
+            Map<String, String> responseHeaders) {
+        return new DisabledJobVertexFlameGraphHandler(leaderRetriever, 
timeout, responseHeaders);
+    }
+
+    private static class DisabledJobVertexFlameGraphHandler
+            extends AbstractRestHandler<
+                    RestfulGateway,
+                    EmptyRequestBody,
+                    JobVertexFlameGraph,
+                    JobVertexFlameGraphParameters> {
+        protected DisabledJobVertexFlameGraphHandler(
+                GatewayRetriever<? extends RestfulGateway> leaderRetriever,
+                Time timeout,
+                Map<String, String> responseHeaders) {
+            super(
+                    leaderRetriever,
+                    timeout,
+                    responseHeaders,
+                    JobVertexFlameGraphHeaders.getInstance());
+        }
+
+        @Override
+        protected CompletableFuture<JobVertexFlameGraph> handleRequest(
+                @Nonnull HandlerRequest<EmptyRequestBody, 
JobVertexFlameGraphParameters> request,
+                @Nonnull RestfulGateway gateway)
+                throws RestHandlerException {
+            return 
CompletableFuture.completedFuture(JobVertexFlameGraph.disabled());
+        }
+    }
 }
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/WebMonitorEndpoint.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/WebMonitorEndpoint.java
index 6794448..1ff0e17 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/WebMonitorEndpoint.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/WebMonitorEndpoint.java
@@ -32,6 +32,7 @@ import 
org.apache.flink.runtime.leaderelection.LeaderElectionService;
 import org.apache.flink.runtime.resourcemanager.ResourceManagerGateway;
 import org.apache.flink.runtime.rest.RestServerEndpoint;
 import org.apache.flink.runtime.rest.RestServerEndpointConfiguration;
+import org.apache.flink.runtime.rest.handler.AbstractRestHandler;
 import org.apache.flink.runtime.rest.handler.RestHandlerConfiguration;
 import org.apache.flink.runtime.rest.handler.RestHandlerSpecification;
 import org.apache.flink.runtime.rest.handler.cluster.ClusterConfigHandler;
@@ -739,8 +740,9 @@ public class WebMonitorEndpoint<T extends RestfulGateway> 
extends RestServerEndp
                         jobVertexBackPressureHandler.getMessageHeaders(),
                         jobVertexBackPressureHandler));
 
+        final AbstractRestHandler<?, ?, ?, ?> jobVertexFlameGraphHandler;
         if (clusterConfiguration.get(RestOptions.ENABLE_FLAMEGRAPH)) {
-            final JobVertexFlameGraphHandler jobVertexFlameGraphHandler =
+            jobVertexFlameGraphHandler =
                     new JobVertexFlameGraphHandler(
                             leaderRetriever,
                             timeout,
@@ -748,11 +750,15 @@ public class WebMonitorEndpoint<T extends RestfulGateway> 
extends RestServerEndp
                             executionGraphCache,
                             executor,
                             initializeThreadInfoTracker(executor));
-            handlers.add(
-                    Tuple2.of(
-                            jobVertexFlameGraphHandler.getMessageHeaders(),
-                            jobVertexFlameGraphHandler));
+        } else {
+            jobVertexFlameGraphHandler =
+                    JobVertexFlameGraphHandler.disabledHandler(
+                            leaderRetriever, timeout, responseHeaders);
         }
+        handlers.add(
+                Tuple2.of(
+                        jobVertexFlameGraphHandler.getMessageHeaders(),
+                        jobVertexFlameGraphHandler));
 
         handlers.add(
                 Tuple2.of(
diff --git 
a/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/threadinfo/JobVertexFlameGraph.java
 
b/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/threadinfo/JobVertexFlameGraph.java
index 4ea94bc..96fe0a2 100644
--- 
a/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/threadinfo/JobVertexFlameGraph.java
+++ 
b/flink-runtime/src/main/java/org/apache/flink/runtime/webmonitor/threadinfo/JobVertexFlameGraph.java
@@ -68,10 +68,21 @@ public class JobVertexFlameGraph implements ResponseBody {
         return "OperatorFlameGraph: endTimestamp=" + endTimestamp + "\n" + 
getRoot().toString();
     }
 
-    public static JobVertexFlameGraph empty() {
+    // Indicates that the task execution has been terminated
+    public static JobVertexFlameGraph terminated() {
         return new JobVertexFlameGraph(-1, null);
     }
 
+    // Indicates that the flame graph feature has been disabled
+    public static JobVertexFlameGraph disabled() {
+        return new JobVertexFlameGraph(-2, null);
+    }
+
+    // Indicates that it is waiting for the first samples to creating the 
flame graph
+    public static JobVertexFlameGraph waiting() {
+        return new JobVertexFlameGraph(-3, null);
+    }
+
     /** Graph node. */
     public static class Node {
 

Reply via email to