public class RestClusterClient<T> extends ClusterClient<T> implements NewClusterClient
ClusterClient
implementation that communicates via HTTP REST requests.ClusterClient.LazyActorSystemLoader
actorSystemLoader, flinkConfig, highAvailabilityServices, lastJobExecutionResult, log, MAX_SLOTS_UNKNOWN, timeout
Constructor and Description |
---|
RestClusterClient(Configuration config,
T clusterId) |
RestClusterClient(Configuration config,
T clusterId,
LeaderRetrievalService webMonitorRetrievalService) |
Modifier and Type | Method and Description |
---|---|
void |
cancel(JobID jobID)
Cancels a job identified by the job id.
|
String |
cancelWithSavepoint(JobID jobId,
String savepointDirectory)
Cancels a job identified by the job id and triggers a savepoint.
|
CompletableFuture<Acknowledge> |
disposeSavepoint(String savepointPath) |
Map<String,OptionalFailure<Object>> |
getAccumulators(JobID jobID,
ClassLoader loader)
Requests and returns the accumulators for the given job identifier.
|
LeaderConnectionInfo |
getClusterConnectionInfo()
Gets the current cluster connection info (may change in case of a HA setup).
|
T |
getClusterId()
Returns the cluster id identifying the cluster to which the client is connected.
|
GetClusterStatusResponse |
getClusterStatus()
Returns the latest cluster status, with number of Taskmanagers and slots.
|
CompletableFuture<JobStatus> |
getJobStatus(JobID jobId)
|
int |
getMaxSlots()
The client may define an upper limit on the number of slots to use.
|
List<String> |
getNewMessages()
May return new messages from the cluster.
|
String |
getWebInterfaceURL()
Returns an URL (as a string) to the JobManager web interface.
|
boolean |
hasUserJarsInClassPath(List<URL> userJarFiles)
Returns true if the client already has the user jar and providing it again would
result in duplicate uploading of the jar.
|
CompletableFuture<Collection<JobStatusMessage>> |
listJobs()
Lists the currently running and finished jobs on the cluster.
|
CompletableFuture<JobResult> |
requestJobResult(JobID jobId)
|
CompletableFuture<Acknowledge> |
rescaleJob(JobID jobId,
int newParallelism)
Rescales the specified job such that it will have the new parallelism.
|
void |
shutdown()
Shuts down the client.
|
void |
shutDownCluster() |
void |
stop(JobID jobID)
Stops a program on Flink cluster whose job-manager is configured in this client's configuration.
|
CompletableFuture<JobSubmissionResult> |
submitJob(JobGraph jobGraph)
Submits the given
JobGraph to the dispatcher. |
JobSubmissionResult |
submitJob(JobGraph jobGraph,
ClassLoader classLoader)
Calls the subclasses' submitJob method.
|
CompletableFuture<String> |
triggerSavepoint(JobID jobId,
String savepointDirectory)
Triggers a savepoint for the job identified by the job id.
|
void |
waitForClusterToBeReady()
Blocks until the client has determined that the cluster is ready for Job submission.
|
connectToJob, endSession, endSessions, getAccumulators, getFlinkConfiguration, getJobGraph, getJobGraph, getJobManagerGateway, getOptimizedPlan, getOptimizedPlan, getOptimizedPlanAsJson, getPrintStatusDuringExecution, isDetached, logAndSysout, retrieveJob, run, run, run, run, run, run, runDetached, setDetached, setPrintStatusDuringExecution
public RestClusterClient(Configuration config, T clusterId) throws Exception
Exception
public RestClusterClient(Configuration config, T clusterId, LeaderRetrievalService webMonitorRetrievalService) throws Exception
Exception
public void shutdown()
ClusterClient
shutdown
in class ClusterClient<T>
public JobSubmissionResult submitJob(JobGraph jobGraph, ClassLoader classLoader) throws ProgramInvocationException
ClusterClient
submitJob
in class ClusterClient<T>
jobGraph
- The JobGraph to be submittedProgramInvocationException
public CompletableFuture<JobStatus> getJobStatus(JobID jobId)
ClusterClient
getJobStatus
in class ClusterClient<T>
public CompletableFuture<JobResult> requestJobResult(@Nonnull JobID jobId)
JobResult
for the given JobID
. The method retries multiple
times to poll the JobResult
before giving up.requestJobResult
in interface NewClusterClient
jobId
- specifying the job for which to retrieve the JobResult
JobResult
once the job has completed or
with a failure if the JobResult
could not be retrieved.public CompletableFuture<JobSubmissionResult> submitJob(@Nonnull JobGraph jobGraph)
JobGraph
to the dispatcher.submitJob
in interface NewClusterClient
jobGraph
- to submitpublic void stop(JobID jobID) throws Exception
ClusterClient
stop
in class ClusterClient<T>
jobID
- the job ID of the streaming program to stopException
- If the job ID is invalid (ie, is unknown or refers to a batch job) or if sending the stop signal
failed. That might be due to an I/O problem, ie, the job-manager is unreachable.public void cancel(JobID jobID) throws Exception
ClusterClient
cancel
in class ClusterClient<T>
jobID
- the job idException
- In case an error occurred.public String cancelWithSavepoint(JobID jobId, @Nullable String savepointDirectory) throws Exception
ClusterClient
cancelWithSavepoint
in class ClusterClient<T>
jobId
- the job idsavepointDirectory
- directory the savepoint should be written toException
- In case an error cocurred.public CompletableFuture<String> triggerSavepoint(JobID jobId, @Nullable String savepointDirectory)
ClusterClient
CheckpointingOptions.SAVEPOINT_DIRECTORY
if it is null.triggerSavepoint
in class ClusterClient<T>
jobId
- job idsavepointDirectory
- directory the savepoint should be written topublic Map<String,OptionalFailure<Object>> getAccumulators(JobID jobID, ClassLoader loader) throws Exception
ClusterClient
getAccumulators
in class ClusterClient<T>
jobID
- The job identifier of a job.loader
- The class loader for deserializing the accumulator results.Exception
public CompletableFuture<Collection<JobStatusMessage>> listJobs()
ClusterClient
listJobs
in class ClusterClient<T>
public T getClusterId()
ClusterClient
getClusterId
in class ClusterClient<T>
public LeaderConnectionInfo getClusterConnectionInfo() throws LeaderRetrievalException
ClusterClient
getClusterConnectionInfo
in class ClusterClient<T>
LeaderRetrievalException
- if the leader could not be retrievedpublic CompletableFuture<Acknowledge> rescaleJob(JobID jobId, int newParallelism)
ClusterClient
rescaleJob
in class ClusterClient<T>
jobId
- specifying the job to modifynewParallelism
- specifying the new parallelism of the rescaled jobpublic CompletableFuture<Acknowledge> disposeSavepoint(String savepointPath)
disposeSavepoint
in class ClusterClient<T>
public void shutDownCluster()
shutDownCluster
in class ClusterClient<T>
public boolean hasUserJarsInClassPath(List<URL> userJarFiles)
ClusterClient
hasUserJarsInClassPath
in class ClusterClient<T>
public void waitForClusterToBeReady()
ClusterClient
This is delayed until right before job submission to report any other errors first (e.g. invalid job definitions/errors in the user jar)
waitForClusterToBeReady
in class ClusterClient<T>
public String getWebInterfaceURL()
ClusterClient
getWebInterfaceURL
in class ClusterClient<T>
public GetClusterStatusResponse getClusterStatus()
ClusterClient
getClusterStatus
in class ClusterClient<T>
public List<String> getNewMessages()
ClusterClient
getNewMessages
in class ClusterClient<T>
public int getMaxSlots()
ClusterClient
getMaxSlots
in class ClusterClient<T>
ClusterClient.MAX_SLOTS_UNKNOWN
) if unknownCopyright © 2014–2020 The Apache Software Foundation. All rights reserved.