Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -23,4 +23,8 @@ public class NotReadyException extends RuntimeException {
public NotReadyException(Exception cause) {
super(cause);
}

public NotReadyException(String message) {
super(message);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@

package org.apache.flink.autoscaler.topology;

import org.apache.flink.autoscaler.exceptions.NotReadyException;
import org.apache.flink.runtime.instance.SlotSharingGroupId;
import org.apache.flink.runtime.jobgraph.JobVertexID;

Expand Down Expand Up @@ -150,6 +151,10 @@ public static JobTopology fromJsonPlan(
ObjectNode plan = objectMapper.readValue(jsonPlan, ObjectNode.class);
ArrayNode nodes = (ArrayNode) plan.get("nodes");

if (nodes == null || nodes.isEmpty()) {
throw new NotReadyException("No nodes found in the plan, job is not ready yet");
}

var vertexInfo = new HashSet<VertexInfo>();

for (JsonNode node : nodes) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@

import org.apache.flink.api.common.JobID;
import org.apache.flink.api.common.JobStatus;
import org.apache.flink.autoscaler.exceptions.NotReadyException;
import org.apache.flink.autoscaler.metrics.FlinkMetric;
import org.apache.flink.autoscaler.metrics.MetricNotFoundException;
import org.apache.flink.autoscaler.topology.IOMetrics;
Expand Down Expand Up @@ -228,6 +229,131 @@ public void testJobTopologyParsingFromJobDetails() throws Exception {
new JobTopology(source, sink), metricsCollector.getJobTopology(jobDetailsInfo));
}

@Test
public void testJobTopologyParsingThrowsNotReadyException() throws Exception {
String s =
"{\n"
+ " \"jid\": \"bb8f15efbb37f2ce519f55cdc0e049bf\",\n"
+ " \"name\": \"State machine job\",\n"
+ " \"isStoppable\": false,\n"
+ " \"state\": \"RUNNING\",\n"
+ " \"start-time\": 1707893512027,\n"
+ " \"end-time\": -1,\n"
+ " \"duration\": 214716,\n"
+ " \"maxParallelism\": -1,\n"
+ " \"now\": 1707893726743,\n"
+ " \"timestamps\": {\n"
+ " \"SUSPENDED\": 0,\n"
+ " \"CREATED\": 1707893512139,\n"
+ " \"FAILING\": 0,\n"
+ " \"FAILED\": 0,\n"
+ " \"INITIALIZING\": 1707893512027,\n"
+ " \"RECONCILING\": 0,\n"
+ " \"RUNNING\": 1707893512217,\n"
+ " \"RESTARTING\": 0,\n"
+ " \"CANCELLING\": 0,\n"
+ " \"FINISHED\": 0,\n"
+ " \"CANCELED\": 0\n"
+ " },\n"
+ " \"vertices\": [\n"
+ " {\n"
+ " \"id\": \"bc764cd8ddf7a0cff126f51c16239658\",\n"
+ " \"name\": \"Source: Custom Source\",\n"
+ " \"maxParallelism\": 128,\n"
+ " \"parallelism\": 2,\n"
+ " \"status\": \"FINISHED\",\n"
+ " \"start-time\": 1707893517277,\n"
+ " \"end-time\": -1,\n"
+ " \"duration\": 209466,\n"
+ " \"tasks\": {\n"
+ " \"DEPLOYING\": 0,\n"
+ " \"INITIALIZING\": 0,\n"
+ " \"SCHEDULED\": 0,\n"
+ " \"CANCELING\": 0,\n"
+ " \"CANCELED\": 0,\n"
+ " \"RECONCILING\": 0,\n"
+ " \"RUNNING\": 2,\n"
+ " \"FAILED\": 0,\n"
+ " \"CREATED\": 0,\n"
+ " \"FINISHED\": 0\n"
+ " },\n"
+ " \"metrics\": {\n"
+ " \"read-bytes\": 0,\n"
+ " \"read-bytes-complete\": true,\n"
+ " \"write-bytes\": 4036982,\n"
+ " \"write-bytes-complete\": true,\n"
+ " \"read-records\": 0,\n"
+ " \"read-records-complete\": true,\n"
+ " \"write-records\": 291629,\n"
+ " \"write-records-complete\": true,\n"
+ " \"accumulated-backpressured-time\": 0,\n"
+ " \"accumulated-idle-time\": 0,\n"
+ " \"accumulated-busy-time\": \"NaN\"\n"
+ " }\n"
+ " },\n"
+ " {\n"
+ " \"id\": \"20ba6b65f97481d5570070de90e4e791\",\n"
+ " \"name\": \"Flat Map -> Sink: Print to Std. Out\",\n"
+ " \"maxParallelism\": 128,\n"
+ " \"parallelism\": 2,\n"
+ " \"status\": \"RUNNING\",\n"
+ " \"start-time\": 1707893517280,\n"
+ " \"end-time\": -1,\n"
+ " \"duration\": 209463,\n"
+ " \"tasks\": {\n"
+ " \"DEPLOYING\": 0,\n"
+ " \"INITIALIZING\": 0,\n"
+ " \"SCHEDULED\": 0,\n"
+ " \"CANCELING\": 0,\n"
+ " \"CANCELED\": 0,\n"
+ " \"RECONCILING\": 0,\n"
+ " \"RUNNING\": 2,\n"
+ " \"FAILED\": 0,\n"
+ " \"CREATED\": 0,\n"
+ " \"FINISHED\": 0\n"
+ " },\n"
+ " \"metrics\": {\n"
+ " \"read-bytes\": 4078629,\n"
+ " \"read-bytes-complete\": true,\n"
+ " \"write-bytes\": 0,\n"
+ " \"write-bytes-complete\": true,\n"
+ " \"read-records\": 291532,\n"
+ " \"read-records-complete\": true,\n"
+ " \"write-records\": 1,\n"
+ " \"write-records-complete\": true,\n"
+ " \"accumulated-backpressured-time\": 0,\n"
+ " \"accumulated-idle-time\": 407702,\n"
+ " \"accumulated-busy-time\": 2\n"
+ " }\n"
+ " }\n"
+ " ],\n"
+ " \"status-counts\": {\n"
+ " \"DEPLOYING\": 0,\n"
+ " \"INITIALIZING\": 0,\n"
+ " \"SCHEDULED\": 0,\n"
+ " \"CANCELING\": 0,\n"
+ " \"CANCELED\": 0,\n"
+ " \"RECONCILING\": 0,\n"
+ " \"RUNNING\": 2,\n"
+ " \"FAILED\": 0,\n"
+ " \"CREATED\": 0,\n"
+ " \"FINISHED\": 0\n"
+ " },\n"
+ " \"plan\": {\n"
+ " \"jid\": \"bb8f15efbb37f2ce519f55cdc0e049bf\",\n"
+ " \"name\": \"State machine job\",\n"
+ " \"type\": \"STREAMING\",\n"
+ " \"nodes\": [\n"
+ " ]\n"
+ " }\n"
+ "}";
JobDetailsInfo jobDetailsInfo = new ObjectMapper().readValue(s, JobDetailsInfo.class);

var metricsCollector = new RestApiMetricsCollector<>();
assertThrows(
NotReadyException.class, () -> metricsCollector.getJobTopology(jobDetailsInfo));
}

@Test
public void testJobTopologyParsingFromJobDetailsWithSlotSharingGroup() throws Exception {
String s =
Expand Down
Loading