public class JobMasterServiceLeadershipRunner extends Object implements JobManagerRunner, LeaderContender
JobMasterServiceProcess
.
The responsibility of this component is to manage the leadership of the JobMasterServiceProcess
. This means that the runner will create an instance of the process when
it obtains the leadership. The process is stopped once the leadership is revoked.
This component only accepts signals (job result completion, initialization failure) as long as it is running and as long as the signals are coming from the current leader process. This ensures that only the current leader can affect this component.
All leadership operations are serialized. This means that granting the leadership has to complete before the leadership can be revoked and vice versa.
The resultFuture
can be completed with the following values: * *
JobManagerRunnerResult
to signal an initialization failure of the JobMasterService
or the completion of a job
Exception
to signal an unexpected failure
Constructor and Description |
---|
JobMasterServiceLeadershipRunner(JobMasterServiceProcessFactory jobMasterServiceProcessFactory,
LeaderElectionService leaderElectionService,
JobResultStore jobResultStore,
LibraryCacheManager.ClassLoaderLease classLoaderLease,
FatalErrorHandler fatalErrorHandler) |
Modifier and Type | Method and Description |
---|---|
CompletableFuture<Acknowledge> |
cancel(Time timeout)
Cancels the currently executed job.
|
CompletableFuture<Void> |
closeAsync()
Trigger the closing of the resource and return the corresponding close future.
|
JobID |
getJobID()
Get the job id of the executed job.
|
CompletableFuture<JobMasterGateway> |
getJobMasterGateway()
Get the
JobMasterGateway of the JobMaster . |
CompletableFuture<JobManagerRunnerResult> |
getResultFuture()
Get the result future of this runner.
|
void |
grantLeadership(UUID leaderSessionID)
Callback method which is called by the
LeaderElectionService upon selecting this
instance as the new leader. |
void |
handleError(Exception exception)
Callback method which is called by
LeaderElectionService in case of an error in the
service thread. |
boolean |
isInitialized()
Flag indicating if the JobManagerRunner has been initialized.
|
CompletableFuture<ExecutionGraphInfo> |
requestJob(Time timeout)
Requests the
ExecutionGraphInfo of the executed job. |
CompletableFuture<JobDetails> |
requestJobDetails(Time timeout)
Request the details of the executed job.
|
CompletableFuture<JobStatus> |
requestJobStatus(Time timeout)
Requests the current job status.
|
void |
revokeLeadership()
Callback method which is called by the
LeaderElectionService upon revoking the
leadership of a former leader. |
void |
start()
Start the execution of the
JobMaster . |
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
close
getDescription
public JobMasterServiceLeadershipRunner(JobMasterServiceProcessFactory jobMasterServiceProcessFactory, LeaderElectionService leaderElectionService, JobResultStore jobResultStore, LibraryCacheManager.ClassLoaderLease classLoaderLease, FatalErrorHandler fatalErrorHandler)
public CompletableFuture<Void> closeAsync()
AutoCloseableAsync
closeAsync
in interface AutoCloseableAsync
public void start() throws Exception
JobManagerRunner
JobMaster
.start
in interface JobManagerRunner
Exception
- if the JobMaster cannot be startedpublic CompletableFuture<JobMasterGateway> getJobMasterGateway()
JobManagerRunner
JobMasterGateway
of the JobMaster
. The future is only completed if
the JobMaster becomes leader.getJobMasterGateway
in interface JobManagerRunner
public CompletableFuture<JobManagerRunnerResult> getResultFuture()
JobManagerRunner
JobMaster
fails. If the
result future is completed exceptionally via JobNotFinishedException
, then this
signals that the job has not been completed successfully. All other exceptional completions
denote an unexpected exception which leads to a process restart.getResultFuture
in interface JobManagerRunner
public JobID getJobID()
JobManagerRunner
getJobID
in interface JobManagerRunner
public CompletableFuture<Acknowledge> cancel(Time timeout)
JobManagerRunner
cancel
in interface JobManagerRunner
timeout
- of this operationpublic CompletableFuture<JobStatus> requestJobStatus(Time timeout)
JobManagerRunner
requestJobStatus
in interface JobManagerRunner
timeout
- for the rpc callpublic CompletableFuture<JobDetails> requestJobDetails(Time timeout)
JobManagerRunner
requestJobDetails
in interface JobManagerRunner
timeout
- for the rpc callpublic CompletableFuture<ExecutionGraphInfo> requestJob(Time timeout)
JobManagerRunner
ExecutionGraphInfo
of the executed job.requestJob
in interface JobManagerRunner
timeout
- for the rpc callExecutionGraphInfo
of the executed jobpublic boolean isInitialized()
JobManagerRunner
isInitialized
in interface JobManagerRunner
public void grantLeadership(UUID leaderSessionID)
LeaderContender
LeaderElectionService
upon selecting this
instance as the new leader. The method is called with the new leader session ID.grantLeadership
in interface LeaderContender
leaderSessionID
- New leader session IDpublic void revokeLeadership()
LeaderContender
LeaderElectionService
upon revoking the
leadership of a former leader. This might happen in case that multiple contenders have been
granted leadership.revokeLeadership
in interface LeaderContender
public void handleError(Exception exception)
LeaderContender
LeaderElectionService
in case of an error in the
service thread.handleError
in interface LeaderContender
exception
- Caught exceptionCopyright © 2014–2023 The Apache Software Foundation. All rights reserved.