Sxnan commented on code in PR #19653: URL: https://github.com/apache/flink/pull/19653#discussion_r877747182
########## flink-runtime/src/main/java/org/apache/flink/runtime/shuffle/ShuffleMaster.java: ########## @@ -84,6 +86,33 @@ CompletableFuture<T> registerPartitionWithProducer( PartitionDescriptor partitionDescriptor, ProducerDescriptor producerDescriptor); + /** + * Returns all the shuffle descriptors for the partitions in the intermediate data set with the + * given id. + * + * @param intermediateDataSetID The id of the intermediate data set. + * @return all the shuffle descriptors for the partitions in the intermediate data set. Null if + * not exist. + */ + default Collection<T> getClusterPartitionShuffleDescriptors( + IntermediateDataSetID intermediateDataSetID) { + return Collections.emptyList(); + } + + /** + * Promote the given partition to cluster partition. + * + * @param shuffleDescriptor The shuffle descriptors of the partition to promote. + */ + default void promotePartition(ShuffleDescriptor shuffleDescriptor) {} + + /** + * Remove the given partition from cluster partition. + * + * @param shuffleDescriptor The shuffle descriptors of the cluster partition to be removed. + */ + default void removeClusterPartition(ShuffleDescriptor shuffleDescriptor) {} Review Comment: Thanks for the comment. You are right that we can call releasePartitionExternally to release the cluster partitions as well. The `removeClusterPartition` method is removed. ########## flink-runtime/src/main/java/org/apache/flink/runtime/shuffle/ShuffleMaster.java: ########## @@ -84,6 +86,33 @@ CompletableFuture<T> registerPartitionWithProducer( PartitionDescriptor partitionDescriptor, ProducerDescriptor producerDescriptor); + /** + * Returns all the shuffle descriptors for the partitions in the intermediate data set with the + * given id. + * + * @param intermediateDataSetID The id of the intermediate data set. + * @return all the shuffle descriptors for the partitions in the intermediate data set. Null if + * not exist. + */ + default Collection<T> getClusterPartitionShuffleDescriptors( Review Comment: By "partition tracker", I assume it means the `JobMasterPartitionTracker`. Currently, the `JobMasterPartitionTracker` is only used to keep track of the partition during job execution and issue release calls to task executors and shuffle masters. It does not keep track of the ShuffleDescriptor of the cluster partition. And its lifecycle is bound to a job. Therefore, it is not suitable for keeping track of the mapping from IntermediateDataSetID to ShuffleDescriptor across job. This method is used by the Job that consumes the cluster partitions produced by some previous jobs to get the Shuffle Descriptors of the cluster partitions. -- 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