WencongLiu commented on code in PR #22341:
URL: https://github.com/apache/flink/pull/22341#discussion_r1264363543


##########
flink-runtime/src/main/java/org/apache/flink/runtime/highavailability/JobResultStore.java:
##########
@@ -43,69 +43,67 @@ public interface JobResultStore {
      * Registers the passed {@link JobResultEntry} instance as {@code dirty} 
which indicates that
      * clean-up operations still need to be performed. Once the job resource 
cleanup has been
      * finalized, we can mark the {@code JobResultEntry} as {@code clean} 
result using {@link
-     * #markResultAsClean(JobID)}.
+     * #markResultAsCleanAsync(JobID)}.
      *
      * @param jobResultEntry The job result we wish to persist.
-     * @throws IOException if the creation of the dirty result failed for IO 
reasons.
-     * @throws IllegalStateException if the passed {@code jobResultEntry} has 
a {@code JobID}
-     *     attached that is already registered in this {@code JobResultStore}.
+     * @return CompletableFuture it the future with {@code true} if the dirty 
result is created
+     *     successfully, otherwise will throw {@link IllegalStateException} if 
the passed {@code
+     *     jobResultEntry} has a {@code JobID} attached that is already 
registered in this {@code
+     *     JobResultStore}.
      */
-    void createDirtyResult(JobResultEntry jobResultEntry) throws IOException, 
IllegalStateException;
+    CompletableFuture<Boolean> createDirtyResultAsync(JobResultEntry 
jobResultEntry);
 
     /**
      * Marks an existing {@link JobResultEntry} as {@code clean}. This 
indicates that no more
      * resource cleanup steps need to be performed. No actions should be 
triggered if the passed
      * {@code JobID} belongs to a job that was already marked as clean.
      *
      * @param jobId Ident of the job we wish to mark as clean.
-     * @throws IOException if marking the {@code dirty} {@code JobResultEntry} 
as {@code clean}
-     *     failed for IO reasons.
-     * @throws NoSuchElementException if there is no corresponding {@code 
dirty} job present in the
+     * @return CompletableFuture is the future with the completed state, which 
will throw {@link
+     *     NoSuchElementException} if there is no corresponding {@code dirty} 
job present in the
      *     store for the given {@code JobID}.
      */
-    void markResultAsClean(JobID jobId) throws IOException, 
NoSuchElementException;
+    CompletableFuture<Void> markResultAsCleanAsync(JobID jobId);
 
     /**
-     * Returns whether the store already contains an entry for a job.
+     * Returns the future of whether the store already contains an entry for a 
job.
      *
      * @param jobId Ident of the job we wish to check the store for.
-     * @return {@code true} if a {@code dirty} or {@code clean} {@link 
JobResultEntry} exists for
-     *     the given {@code JobID}; otherwise {@code false}.
-     * @throws IOException if determining whether a job entry is present in 
the store failed for IO
-     *     reasons.
+     * @return CompletableFuture with {@code true} if a {@code dirty} or 
{@code clean} {@link
+     *     JobResultEntry} exists for the given {@code JobID}; otherwise 
{@code false}.
      */
-    default boolean hasJobResultEntry(JobID jobId) throws IOException {
-        return hasDirtyJobResultEntry(jobId) || hasCleanJobResultEntry(jobId);
+    default CompletableFuture<Boolean> hasJobResultEntryAsync(JobID jobId) {
+        return hasDirtyJobResultEntryAsync(jobId)
+                .thenCombine(
+                        hasCleanJobResultEntryAsync(jobId),
+                        (result1, result2) -> result1 || result2);
     }
 
     /**
-     * Returns whether the store already contains a {@code dirty} entry for 
the given {@code JobID}.
+     * Returns the future of whether the store contains a {@code dirty} entry 
for the given {@code
+     * JobID}.
      *
      * @param jobId Ident of the job we wish to check the store for.
-     * @return {@code true}, if a {@code dirty} entry exists for the given 
{@code JobID}; otherwise
-     *     {@code false}.
-     * @throws IOException if determining whether a job entry is present in 
the store failed for IO
-     *     reasons.
+     * @return CompletableFuture with value of {@code true}, if a {@code 
dirty} entry exists for the
+     *     given {@code JobID}; otherwise Completable with value of {@code 
false}.
      */
-    boolean hasDirtyJobResultEntry(JobID jobId) throws IOException;
+    CompletableFuture<Boolean> hasDirtyJobResultEntryAsync(JobID jobId);
 
     /**
-     * Returns whether the store already contains a {@code clean} entry for 
the given {@code JobID}.
+     * Returns the future of whether the store contains a {@code clean} entry 
for the given {@code
+     * JobID}.
      *
      * @param jobId Ident of the job we wish to check the store for.
-     * @return {@code true}, if a {@code clean} entry exists for the given 
{@code JobID}; otherwise
-     *     {@code false}.
-     * @throws IOException if determining whether a job entry is present in 
the store failed for IO
-     *     reasons.
+     * @return CompletableFuture with value of {@code true}, if a {@code 
clean} entry exists for the
+     *     given {@code JobID}; otherwise Completable with value of {@code 
false}.
      */
-    boolean hasCleanJobResultEntry(JobID jobId) throws IOException;
+    CompletableFuture<Boolean> hasCleanJobResultEntryAsync(JobID jobId);
 
     /**
-     * Get the persisted {@link JobResult} instances that are marked as {@code 
dirty}. This is
-     * useful for recovery of finalization steps.
+     * Returns the future of persisted {@link JobResult} instances that are 
marked as {@code dirty}.
+     * This is useful for recovery of finalization steps.
      *
-     * @return A set of dirty {@code JobResults} from the store.
-     * @throws IOException if collecting the set of dirty results failed for 
IO reasons.
+     * @return CompletableFuture with value of a set of dirty {@code 
JobResults} from the store.
      */
-    Set<JobResult> getDirtyResults() throws IOException;
+    CompletableFuture<Set<JobResult>> getDirtyResultsAsync();

Review Comment:
   Currently there are two methods `getDirtyResults ` and `getDirtyResultsAsync 
`.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to