Interface JobResultStore

  • All Known Implementing Classes:
    AbstractThreadsafeJobResultStore, EmbeddedJobResultStore, FileSystemJobResultStore

    @Internal
    public interface JobResultStore
    A 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 Detail

      • createDirtyResult

        void createDirtyResult​(JobResultEntry jobResultEntry)
                        throws IOException,
                               IllegalStateException
        Registers the passed JobResultEntry instance as dirty which indicates that clean-up operations still need to be performed. Once the job resource cleanup has been finalized, we can mark the JobResultEntry as clean result using markResultAsClean(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 passed jobResultEntry has a JobID attached that is already registered in this JobResultStore.
      • markResultAsClean

        void markResultAsClean​(org.apache.flink.api.common.JobID jobId)
                        throws IOException,
                               NoSuchElementException
        Marks an existing JobResultEntry as clean. This indicates that no more resource cleanup steps need to be performed. No actions should be triggered if the passed JobID belongs 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 the dirty JobResultEntry as clean failed for IO reasons.
        NoSuchElementException - if there is no corresponding dirty job present in the store for the given JobID.
      • hasJobResultEntry

        default boolean hasJobResultEntry​(org.apache.flink.api.common.JobID jobId)
                                   throws IOException
        Returns whether the store already contains an entry for a job.
        Parameters:
        jobId - Ident of the job we wish to check the store for.
        Returns:
        true if a dirty or clean JobResultEntry exists for the given JobID; otherwise false.
        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 IOException
        Returns whether the store already contains a dirty entry for the given JobID.
        Parameters:
        jobId - Ident of the job we wish to check the store for.
        Returns:
        true, if a dirty entry exists for the given JobID; otherwise false.
        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 IOException
        Returns whether the store already contains a clean entry for the given JobID.
        Parameters:
        jobId - Ident of the job we wish to check the store for.
        Returns:
        true, if a clean entry exists for the given JobID; otherwise false.
        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 persisted JobResult instances that are marked as dirty. This is useful for recovery of finalization steps.
        Returns:
        A set of dirty JobResults from the store.
        Throws:
        IOException - if collecting the set of dirty results failed for IO reasons.