Interface JobResultStore
-
- All Known Implementing Classes:
AbstractThreadsafeJobResultStore,EmbeddedJobResultStore,FileSystemJobResultStore
@Internal public interface JobResultStoreA storage for the results of globally terminated jobs. These results can have the following states:dirty- indicating that the corresponding job is not properly cleaned up, yet.clean- indicating that the cleanup of the corresponding job is performed and no further actions need to be applied.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description voidcreateDirtyResult(JobResultEntry jobResultEntry)Registers the passedJobResultEntryinstance asdirtywhich indicates that clean-up operations still need to be performed.Set<JobResult>getDirtyResults()Get the persistedJobResultinstances that are marked asdirty.booleanhasCleanJobResultEntry(org.apache.flink.api.common.JobID jobId)Returns whether the store already contains acleanentry for the givenJobID.booleanhasDirtyJobResultEntry(org.apache.flink.api.common.JobID jobId)Returns whether the store already contains adirtyentry for the givenJobID.default booleanhasJobResultEntry(org.apache.flink.api.common.JobID jobId)Returns whether the store already contains an entry for a job.voidmarkResultAsClean(org.apache.flink.api.common.JobID jobId)Marks an existingJobResultEntryasclean.
-
-
-
Method Detail
-
createDirtyResult
void createDirtyResult(JobResultEntry jobResultEntry) throws IOException, IllegalStateException
Registers the passedJobResultEntryinstance asdirtywhich indicates that clean-up operations still need to be performed. Once the job resource cleanup has been finalized, we can mark theJobResultEntryascleanresult usingmarkResultAsClean(JobID).- Parameters:
jobResultEntry- The job result we wish to persist.- Throws:
IOException- if the creation of the dirty result failed for IO reasons.IllegalStateException- if the passedjobResultEntryhas aJobIDattached that is already registered in thisJobResultStore.
-
markResultAsClean
void markResultAsClean(org.apache.flink.api.common.JobID jobId) throws IOException, NoSuchElementExceptionMarks an existingJobResultEntryasclean. This indicates that no more resource cleanup steps need to be performed. No actions should be triggered if the passedJobIDbelongs to a job that was already marked as clean.- Parameters:
jobId- Ident of the job we wish to mark as clean.- Throws:
IOException- if marking thedirtyJobResultEntryascleanfailed for IO reasons.NoSuchElementException- if there is no correspondingdirtyjob present in the store for the givenJobID.
-
hasJobResultEntry
default boolean hasJobResultEntry(org.apache.flink.api.common.JobID jobId) throws IOExceptionReturns whether the store already contains an entry for a job.- Parameters:
jobId- Ident of the job we wish to check the store for.- Returns:
trueif adirtyorcleanJobResultEntryexists for the givenJobID; otherwisefalse.- Throws:
IOException- if determining whether a job entry is present in the store failed for IO reasons.
-
hasDirtyJobResultEntry
boolean hasDirtyJobResultEntry(org.apache.flink.api.common.JobID jobId) throws IOExceptionReturns whether the store already contains adirtyentry for the givenJobID.- Parameters:
jobId- Ident of the job we wish to check the store for.- Returns:
true, if adirtyentry exists for the givenJobID; otherwisefalse.- Throws:
IOException- if determining whether a job entry is present in the store failed for IO reasons.
-
hasCleanJobResultEntry
boolean hasCleanJobResultEntry(org.apache.flink.api.common.JobID jobId) throws IOExceptionReturns whether the store already contains acleanentry for the givenJobID.- Parameters:
jobId- Ident of the job we wish to check the store for.- Returns:
true, if acleanentry exists for the givenJobID; otherwisefalse.- Throws:
IOException- if determining whether a job entry is present in the store failed for IO reasons.
-
getDirtyResults
Set<JobResult> getDirtyResults() throws IOException
Get the persistedJobResultinstances that are marked asdirty. This is useful for recovery of finalization steps.- Returns:
- A set of dirty
JobResultsfrom the store. - Throws:
IOException- if collecting the set of dirty results failed for IO reasons.
-
-