lsyldliu commented on code in PR #20223:
URL: https://github.com/apache/flink/pull/20223#discussion_r916762924
##########
flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/DefaultExecutionGraph.java:
##########
@@ -1544,6 +1552,30 @@ private void notifyJobStatusChange(JobStatus newState) {
}
}
+ private void notifyJobStatusHooks(JobStatus newState, Throwable cause) {
+ JobID jobID = jobInformation.getJobId();
+ for (JobStatusHook hook : jobStatusHooks) {
+ try {
+ switch (newState) {
+ case CREATED:
+ hook.onCreated(jobID);
+ break;
+ case CANCELED:
+ hook.onCanceled(jobID);
+ break;
+ case FAILED:
+ hook.onFailed(jobID, cause);
+ break;
+ case FINISHED:
+ hook.onFinished(jobID);
+ break;
+ }
+ } catch (Throwable ignore) {
+ LOG.warn("Error while notifying JobStatusHook[{}]",
hook.getClass(), ignore);
Review Comment:
I think this exception should not be ignored, if the hook execute failed,
the job execution will occur other exception which will confuse the user, so I
think we should throw this exception early.
##########
flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/JobStatusHook.java:
##########
@@ -0,0 +1,41 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.flink.runtime.executiongraph;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.api.common.JobID;
+
+import java.io.Serializable;
+
+/** Hooks provided by users on job status changing. */
Review Comment:
I suggest add more annotation about this hook, let user know its function
and how to use it.
##########
flink-runtime/src/main/java/org/apache/flink/runtime/executiongraph/JobStatusHook.java:
##########
@@ -0,0 +1,41 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.flink.runtime.executiongraph;
+
+import org.apache.flink.annotation.Internal;
+import org.apache.flink.api.common.JobID;
+
+import java.io.Serializable;
+
+/** Hooks provided by users on job status changing. */
+@Internal
+public interface JobStatusHook extends Serializable {
+
+ /** When Job become CREATED status. It would only be called one time. */
Review Comment:
```suggestion
/** When Job become CREATED status, it would only be called one time. */
```
##########
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamingJobGraphGenerator.java:
##########
@@ -225,6 +225,8 @@ private JobGraph createJobGraph() {
setVertexDescription();
+ jobGraph.setJobStatusHooks(streamGraph.getJobStatusHooks());
Review Comment:
I think it would be better to set hooks when hooks > 0.
##########
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/graph/StreamGraph.java:
##########
@@ -1034,4 +1037,18 @@ public void setVertexNameIncludeIndexPrefix(boolean
includePrefix) {
public boolean isVertexNameIncludeIndexPrefix() {
return this.vertexNameIncludeIndexPrefix;
}
+
+ /** Registers the JobStatusHook. */
+ public void registerJobStatusHook(JobStatusHook hook) {
+ if (hook == null) {
+ throw new IllegalArgumentException();
Review Comment:
Give an explicitly exception message.
##########
flink-runtime/src/main/java/org/apache/flink/runtime/jobgraph/JobGraph.java:
##########
@@ -628,4 +632,13 @@ public void writeUserArtifactEntriesToConfiguration() {
userArtifact.getKey(), userArtifact.getValue(),
jobConfiguration);
}
}
+
+ public void setJobStatusHooks(List<JobStatusHook> hooks) {
Review Comment:
I think here two implementation way:
1. follow `addJars` method, method rename to `addJobStatusHooks `, add the
object of list to existing list. Maybe we can register hooks multiply instead
of clear it before set.
2. follow `setClasspaths` method, jobStatusHooks initial value is
`Collection.emptyList()`, here assign list to it.
--
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]