shibd commented on code in PR #22860: URL: https://github.com/apache/pulsar/pull/22860#discussion_r1630435251
########## pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java: ########## @@ -1283,7 +1288,7 @@ private CompletableFuture<Optional<Topic>> createNonPersistentTopic(String topic }).exceptionally(ex -> { log.warn("Replication check failed. Removing topic from topics list {}, {}", topic, ex.getCause()); nonPersistentTopic.stopReplProducers().whenComplete((v, exception) -> { - pulsar.getExecutor().execute(() -> topics.remove(topic, topicFuture)); + pulsar.getExecutor().execute(() -> topics.remove(topic)); Review Comment: I have not found any reason for it. This approach has been the same from the beginning, tracing back to the `initial import`. It seems that ever PR just continued with the same style to maintain consistency. https://github.com/apache/pulsar/pull/538/files#diff-0210356c8a88e4efa89eb769a027fa6c166db479dbad8bbbbc704c6ed6e317f5R463 However, in reality, if you want to compare topic futures, it is easy to make mistakes in deletion. For examples: ``` java static CompletableFuture<Optional<Integer>> m1 = CompletableFuture.completedFuture(Optional.of(100)); static CompletableFuture<Optional<String>> m2 = CompletableFuture.completedFuture(Optional.of("value1")); public static void main(String[] args) throws ExecutionException, InterruptedException { ConcurrentOpenHashMap<String, CompletableFuture<Optional<String>>> maps = ConcurrentOpenHashMap.<String, CompletableFuture<Optional<String>>>newBuilder() .build(); maps.computeIfAbsent("test1", (key) -> { return getM1().thenCompose(__ -> { return getM2(); }); }); System.out.println("debug m1" + m1 + "@" + System.identityHashCode(m1)); System.out.println("debug m2" + m2 + "@" + System.identityHashCode(m2)); CompletableFuture<Optional<String>> test1 = maps.get("test1"); System.out.println("debug1: " + test1.get().get()); System.out.println("debug test1:" + test1 + "@" + System.identityHashCode(test1)); maps.remove("test1", m2); CompletableFuture<Optional<String>> test2 = maps.get("test1"); System.out.println("debug2: " + test2.get().get()); System.out.println("debug test2:" + test2 + "@" + System.identityHashCode(test2)); } public static CompletableFuture<Optional<Integer>> getM1() { return m1; } public static CompletableFuture<Optional<String>> getM2() { return m2; } ``` A similar code was written here. So, what's in the cache is not the original `topicFuture`, its a transformed future. https://github.com/apache/pulsar/blob/21596227415968dd9b9219779657ff74decfa45b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/BrokerService.java#L1042-L1073 -- 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: commits-unsubscr...@pulsar.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org