kamalcph commented on code in PR #13947:
URL: https://github.com/apache/kafka/pull/13947#discussion_r1296356436


##########
core/src/main/java/kafka/log/remote/RemoteLogManager.java:
##########
@@ -343,21 +345,80 @@ public void onLeadershipChange(Set<Partition> 
partitionsBecomeLeader,
     /**
      * Deletes the internal topic partition info if delete flag is set as true.
      *
-     * @param topicPartition topic partition to be stopped.
+     * @param topicPartitions topic partitions that needs to be stopped.
      * @param delete         flag to indicate whether the given topic 
partitions to be deleted or not.
+     * @param errorHandler   callback to handle any errors while stopping the 
partitions.
      */
-    public void stopPartitions(TopicPartition topicPartition, boolean delete) {
+    public void stopPartitions(Set<TopicPartition> topicPartitions,
+                               boolean delete,
+                               BiConsumer<TopicPartition, Throwable> 
errorHandler) {
+        LOGGER.debug("Stopping {} partitions, delete: {}", 
topicPartitions.size(), delete);
+        Set<TopicIdPartition> topicIdPartitions = topicPartitions.stream()
+                .filter(topicIdByPartitionMap::containsKey)
+                .map(tp -> new TopicIdPartition(topicIdByPartitionMap.get(tp), 
tp))
+                .collect(Collectors.toSet());
+
+        topicIdPartitions.forEach(tpId -> {
+            try {
+                RLMTaskWithFuture task = leaderOrFollowerTasks.remove(tpId);
+                if (task != null) {
+                    LOGGER.info("Cancelling the RLM task for tpId: {}", tpId);
+                    task.cancel();
+                }
+                if (delete) {
+                    LOGGER.info("Deleting the remote log segments task for 
partition: {}", tpId);
+                    deleteRemoteLogPartition(tpId);
+                }
+            } catch (Exception ex) {
+                errorHandler.accept(tpId.topicPartition(), ex);
+                LOGGER.error("Error while stopping the partition: {}, delete: 
{}", tpId.topicPartition(), delete, ex);
+            }
+        });
+
         if (delete) {
-            // Delete from internal datastructures only if it is to be deleted.
-            Uuid topicIdPartition = topicPartitionIds.remove(topicPartition);
-            LOGGER.debug("Removed partition: {} from topicPartitionIds", 
topicIdPartition);
+            // NOTE: this#stopPartitions method is called when Replica state 
changes to Offline and ReplicaDeletionStarted
+            remoteLogMetadataManager.onStopPartitions(topicIdPartitions);
+            topicPartitions.forEach(topicIdByPartitionMap::remove);
+        }
+    }
+
+    private void deleteRemoteLogPartition(TopicIdPartition partition) throws 
RemoteStorageException, ExecutionException, InterruptedException {
+        List<RemoteLogSegmentMetadata> metadataList = new ArrayList<>();
+        
remoteLogMetadataManager.listRemoteLogSegments(partition).forEachRemaining(metadataList::add);
+
+        List<RemoteLogSegmentMetadataUpdate> deleteSegmentStartedEvents = 
metadataList.stream()
+                .map(metadata ->
+                        new 
RemoteLogSegmentMetadataUpdate(metadata.remoteLogSegmentId(), 
time.milliseconds(),
+                                metadata.customMetadata(), 
RemoteLogSegmentState.DELETE_SEGMENT_STARTED, brokerId))
+                .collect(Collectors.toList());
+        publishEvents(deleteSegmentStartedEvents).get();

Review Comment:
   We are taking the `replicaStateChangeLock` in the ReplicaManager. This 
method is called only by the request handler threads to handle the 
stop-partition so far.



-- 
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: jira-unsubscr...@kafka.apache.org

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

Reply via email to