1996fanrui commented on code in PR #21999:
URL: https://github.com/apache/flink/pull/21999#discussion_r1115424478
##########
flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskITCase.java:
##########
@@ -110,6 +118,47 @@ public void testFailInEndOfConstructor() throws Exception {
.build(EXECUTOR_RESOURCE.getExecutor()));
}
+ @Test
+ public void
streamTaskAsyncExceptionHandlerHandleExceptionInInitializingDuration()
Review Comment:
There are some issues about the method name:
1. Test method name should start with `test`.
2. It's too long, and the Class Name is about `StreamTask`, so I think the
`StreamTask` can be removed for the method name.
3. Rename the `Initializing` to `restoring`
How about `testAsyncExceptionCanBeHandledDuringRestoring`?
##########
flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskITCase.java:
##########
@@ -110,6 +118,47 @@ public void testFailInEndOfConstructor() throws Exception {
.build(EXECUTOR_RESOURCE.getExecutor()));
}
+ @Test
+ public void
streamTaskAsyncExceptionHandlerHandleExceptionInInitializingDuration()
+ throws Exception {
+ MockEnvironment mockEnvironment = MockEnvironment.builder().build();
+ RuntimeException expectedException = new RuntimeException("RUNTIME
EXCEPTION");
+
+
mockEnvironment.setExpectedExternalFailureCause(AsynchronousException.class);
+ final String expectedErrorMessage = "EXPECTED_ERROR MESSAGE";
+
+ NoOpStreamTask noOpStreamTask =
+ new NoOpStreamTask(mockEnvironment) {
+ volatile boolean handledAsyncException = false;
+
+ @Override
+ protected void init() throws Exception {
+ super.init();
+
+ CompletableFuture.runAsync(
+ () -> {
+ this.handleAsyncException(
+ expectedErrorMessage,
expectedException);
+ handledAsyncException = true;
+ });
+ while (!handledAsyncException) {
+ ThreadUtils.sleep(Duration.ofMillis(10));
+ }
+ }
+ };
+ noOpStreamTask.invoke();
Review Comment:
How about call the `noOpStreamTask.restore()` due to this test checks
whether AsyncException can be handled during restoring?
Please correct me, if `noOpStreamTask.restore()` doesn't work here.
##########
flink-streaming-java/src/test/java/org/apache/flink/streaming/runtime/tasks/StreamTaskITCase.java:
##########
@@ -110,6 +118,47 @@ public void testFailInEndOfConstructor() throws Exception {
.build(EXECUTOR_RESOURCE.getExecutor()));
}
+ @Test
+ public void
streamTaskAsyncExceptionHandlerHandleExceptionInInitializingDuration()
+ throws Exception {
+ MockEnvironment mockEnvironment = MockEnvironment.builder().build();
+ RuntimeException expectedException = new RuntimeException("RUNTIME
EXCEPTION");
+
+
mockEnvironment.setExpectedExternalFailureCause(AsynchronousException.class);
+ final String expectedErrorMessage = "EXPECTED_ERROR MESSAGE";
+
+ NoOpStreamTask noOpStreamTask =
+ new NoOpStreamTask(mockEnvironment) {
+ volatile boolean handledAsyncException = false;
+
+ @Override
+ protected void init() throws Exception {
+ super.init();
+
+ CompletableFuture.runAsync(
Review Comment:
Add a comment : `Throw exception during restoring.`
--
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: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]