[ 
https://issues.apache.org/jira/browse/FLINK-21928?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17308739#comment-17308739
 ] 

Till Rohrmann commented on FLINK-21928:
---------------------------------------

What would be the desired return value for the client as the response to the 
submit call? If we say that we have the job already executed then we might not 
have the final result of the job which we can tell the client?

In general, I think the {{RunningJobsRegistry}} does not properly work. The 
underlying problem is that the registry needs to contain information which can 
outlive the lifetime of a {{Dispatcher}} in order to work. See FLINK-11813 for 
more details. 

One idea to fix it could be that the registry is owned by the deployer of a 
Flink cluster and, hence, the cluster no longer clears the entries of the 
registry. The user of the Flink cluster is then responsible for cleaning up the 
registry after the cluster has been shut down. Moreover, we could say that this 
feature needs to be explicitly activated and is actually only useful for 
guarding against standby {{JobManagers}} which should not execute the job after 
it has been completed by another {{JobManager}}.

As a kind of a quick fix, we could think about simply removing the registry 
since it is broken anyway.

> DuplicateJobSubmissionException after JobManager failover
> ---------------------------------------------------------
>
>                 Key: FLINK-21928
>                 URL: https://issues.apache.org/jira/browse/FLINK-21928
>             Project: Flink
>          Issue Type: Bug
>          Components: Runtime / Coordination
>    Affects Versions: 1.10.3, 1.11.3, 1.12.2
>         Environment: StandaloneApplicationClusterEntryPoint using a fixed job 
> ID, High Availability enabled
>            Reporter: Ufuk Celebi
>            Priority: Major
>
> Consider the following scenario:
>  * Environment: StandaloneApplicationClusterEntryPoint using a fixed job ID, 
> high availability enabled
>  * Flink job reaches a globally terminal state
>  * Flink job is marked as finished in the high-availability service's 
> RunningJobsRegistry
>  * The JobManager fails over
> On recovery, the [Dispatcher throws DuplicateJobSubmissionException, because 
> the job is marked as done in the 
> RunningJobsRegistry|https://github.com/apache/flink/blob/release-1.12.2/flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/Dispatcher.java#L332-L340].
> When this happens, users cannot get out of the situation without manually 
> redeploying the JobManager process and changing the job ID^1^.
> The desired semantics are that we don't want to re-execute a job that has 
> reached a globally terminal state. In this particular case, we know that the 
> job has already reached such a state (as it has been marked in the registry). 
> Therefore, we could handle this case by executing the regular termination 
> sequence instead of throwing a DuplicateJobSubmission.
> ---
> ^1^ With ZooKeeper HA, the respective node is not ephemeral. In Kubernetes 
> HA, there is no  notion of ephemeral data that is tied to a session in the 
> first place afaik.
>  



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to