fapaul commented on a change in pull request #17995: URL: https://github.com/apache/flink/pull/17995#discussion_r762948703
########## File path: flink-runtime/src/test/java/org/apache/flink/runtime/dispatcher/DispatcherTest.java ########## @@ -900,6 +901,41 @@ public void testJobDataAreCleanedUpInCorrectOrderOnFailedJob() throws Exception testJobDataAreCleanedUpInCorrectOrder(JobStatus.FAILED); } + @Test + public void testRetrieveJobResultAfterSubmissionOfFailedJob() throws Exception { + dispatcher = + createAndStartDispatcher( + heartbeatServices, + haServices, + new ExpectedJobIdJobManagerRunnerFactory( + jobId, createdJobManagerRunnerLatch)); + + final DispatcherGateway dispatcherGateway = + dispatcher.getSelfGateway(DispatcherGateway.class); + + final JobID failedJobId = new JobID(); + + final String jobName = "Failed Streaming Job"; + dispatcherGateway.submitFailedJob( + new JobStartupFailedException( + failedJobId, jobName, Collections.singletonList(() -> ""))); + + final ExecutionGraphInfo executionGraphInfo = + dispatcherGateway.requestExecutionGraphInfo(failedJobId, TIMEOUT).get(); + assertEquals(executionGraphInfo.getJobId(), failedJobId); + final ArchivedExecutionGraph archivedExecutionGraph = + executionGraphInfo.getArchivedExecutionGraph(); + assertEquals(archivedExecutionGraph.getState(), JobStatus.FAILED); + assertEquals(archivedExecutionGraph.getJobName(), jobName); + assertThat( + archivedExecutionGraph + .getFailureInfo() + .getException() + .deserializeError(ClassLoader.getSystemClassLoader()) + .toString(), + containsString(JobStartupFailedException.class.getName())); Review comment: I struggled a bit with this Assertion because it seems there is no way to deserialize the `SerializedException` back to the JobStartupFailedException. Even after calling deserialize, it is still a `SerializedException`. Therefore I used `toString` to mitigate it but I agree it is not an elegant solution. -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org