dmvk commented on a change in pull request #18749:
URL: https://github.com/apache/flink/pull/18749#discussion_r806506151



##########
File path: 
flink-runtime/src/main/java/org/apache/flink/runtime/dispatcher/cleanup/DispatcherResourceCleanerFactory.java
##########
@@ -108,6 +108,7 @@ public ResourceCleaner createGlobalResourceCleaner(
                 .withRegularCleanup(jobGraphWriter)
                 .withRegularCleanup(blobServer)
                 .withRegularCleanup(highAvailabilityServices)
+                .withRegularCleanup(jobManagerMetricGroup)

Review comment:
       OT: typo in the `createExponentialRetryStategy` method below

##########
File path: 
flink-runtime/src/main/java/org/apache/flink/runtime/metrics/groups/JobManagerMetricGroup.java
##########
@@ -38,7 +39,7 @@
  * tasks any more
  */
 public class JobManagerMetricGroup extends 
ComponentMetricGroup<JobManagerMetricGroup>
-        implements LocallyCleanableResource {
+        implements LocallyCleanableResource, GloballyCleanableResource {

Review comment:
       This doesn't feel right. As long as this class doesn't have any 
artifacts that can outlive the JM, it shouldn't implement this interface.
   
   How about doing something like this instead?
   
   ```java
       @Override
       public ResourceCleaner createGlobalResourceCleaner(
               ComponentMainThreadExecutor mainThreadExecutor) {
   
           return DefaultResourceCleaner.forGloballyCleanableResources(
                           mainThreadExecutor, cleanupExecutor, retryStrategy)
                   .withPrioritizedCleanup(jobManagerRunnerRegistry)
                   .withRegularCleanup(jobGraphWriter)
                   .withRegularCleanup(blobServer)
                   .withRegularCleanup(highAvailabilityServices)
                   .withRegularCleanup(ofLocalResource(jobManagerMetricGroup))
                   .build();
       }
   
       /**
        * A simple wrapper for the resources that don't have any artifacts that 
can outlive the {@link
        * org.apache.flink.runtime.dispatcher.Dispatcher}, but we still want to 
clean up their local
        * state when we terminate locally.
        *
        * @param localResource Local resource that we want to clean during a 
global cleanup.
        * @return Globally cleanable resource.
        */
       private static GloballyCleanableResource ofLocalResource(
               LocallyCleanableResource localResource) {
           return localResource::localCleanupAsync;
       }




-- 
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: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


Reply via email to