|
37 | 37 | import java.nio.charset.StandardCharsets; |
38 | 38 | import java.util.ArrayList; |
39 | 39 | import java.util.Collection; |
| 40 | +import java.util.Collections; |
| 41 | +import java.util.HashSet; |
40 | 42 | import java.util.List; |
| 43 | +import java.util.Set; |
41 | 44 | import java.util.UUID; |
42 | 45 | import java.util.concurrent.CompletableFuture; |
43 | 46 | import java.util.concurrent.CountDownLatch; |
|
46 | 49 | import com.google.common.collect.Multimap; |
47 | 50 | import com.google.common.collect.Sets; |
48 | 51 | import java.util.concurrent.atomic.AtomicBoolean; |
| 52 | +import java.util.function.Supplier; |
49 | 53 | import lombok.Cleanup; |
50 | 54 | import org.apache.bookkeeper.client.LedgerHandle; |
| 55 | +import org.apache.bookkeeper.mledger.ManagedCursor; |
51 | 56 | import org.apache.bookkeeper.mledger.ManagedLedger; |
52 | 57 | import org.apache.pulsar.broker.service.BrokerService; |
53 | 58 | import org.apache.pulsar.broker.service.BrokerTestBase; |
54 | 59 | import org.apache.pulsar.broker.stats.PrometheusMetricsTest; |
55 | 60 | import org.apache.pulsar.broker.stats.prometheus.PrometheusMetricsGenerator; |
56 | | -import org.apache.pulsar.client.api.*; |
| 61 | +import org.apache.pulsar.client.api.Consumer; |
| 62 | +import org.apache.pulsar.client.api.Message; |
| 63 | +import org.apache.pulsar.client.api.MessageId; |
| 64 | +import org.apache.pulsar.client.api.MessageListener; |
| 65 | +import org.apache.pulsar.client.api.MessageRoutingMode; |
| 66 | +import org.apache.pulsar.client.api.Producer; |
| 67 | +import org.apache.pulsar.client.api.PulsarClient; |
| 68 | +import org.apache.pulsar.client.api.PulsarClientException; |
| 69 | +import org.apache.pulsar.client.api.Schema; |
| 70 | +import org.apache.pulsar.client.api.SubscriptionType; |
57 | 71 | import org.apache.pulsar.common.naming.NamespaceBundle; |
58 | 72 | import org.apache.pulsar.common.naming.TopicName; |
| 73 | +import org.apache.pulsar.common.policies.data.ClusterData; |
59 | 74 | import org.apache.pulsar.common.policies.data.Policies; |
| 75 | +import org.apache.pulsar.common.policies.data.TenantInfo; |
60 | 76 | import org.apache.pulsar.common.policies.data.TopicStats; |
61 | 77 | import org.awaitility.Awaitility; |
62 | 78 | import org.junit.Assert; |
@@ -402,4 +418,59 @@ public void testDeleteTopicFail() throws Exception { |
402 | 418 | makeDeletedFailed.set(false); |
403 | 419 | persistentTopic.delete().get(); |
404 | 420 | } |
| 421 | + |
| 422 | + @DataProvider(name = "topicLevelPolicy") |
| 423 | + public static Object[][] topicLevelPolicy() { |
| 424 | + return new Object[][] { { true }, { false } }; |
| 425 | + } |
| 426 | + |
| 427 | + @Test(dataProvider = "topicLevelPolicy") |
| 428 | + public void testCreateTopicWithZombieReplicatorCursor(boolean topicLevelPolicy) throws Exception { |
| 429 | + final String namespace = "prop/ns-abc"; |
| 430 | + final String topicName = "persistent://" + namespace |
| 431 | + + "/testCreateTopicWithZombieReplicatorCursor" + topicLevelPolicy; |
| 432 | + final String remoteCluster = "remote"; |
| 433 | + admin.topics().createNonPartitionedTopic(topicName); |
| 434 | + admin.topics().createSubscription(topicName, conf.getReplicatorPrefix() + "." + remoteCluster, |
| 435 | + MessageId.earliest, true); |
| 436 | + |
| 437 | + admin.clusters().createCluster(remoteCluster, ClusterData.builder() |
| 438 | + .serviceUrl("http://localhost:11112") |
| 439 | + .brokerServiceUrl("pulsar://localhost:11111") |
| 440 | + .build()); |
| 441 | + TenantInfo tenantInfo = admin.tenants().getTenantInfo("prop"); |
| 442 | + tenantInfo.getAllowedClusters().add(remoteCluster); |
| 443 | + admin.tenants().updateTenant("prop", tenantInfo); |
| 444 | + |
| 445 | + if (topicLevelPolicy) { |
| 446 | + admin.topics().setReplicationClusters(topicName, Collections.singletonList(remoteCluster)); |
| 447 | + } else { |
| 448 | + admin.namespaces().setNamespaceReplicationClustersAsync( |
| 449 | + namespace, Collections.singleton(remoteCluster)).get(); |
| 450 | + } |
| 451 | + |
| 452 | + final PersistentTopic topic = (PersistentTopic) pulsar.getBrokerService().getTopic(topicName, false) |
| 453 | + .get(3, TimeUnit.SECONDS).orElse(null); |
| 454 | + assertNotNull(topic); |
| 455 | + |
| 456 | + final Supplier<Set<String>> getCursors = () -> { |
| 457 | + final Set<String> cursors = new HashSet<>(); |
| 458 | + final Iterable<ManagedCursor> iterable = topic.getManagedLedger().getCursors(); |
| 459 | + iterable.forEach(c -> cursors.add(c.getName())); |
| 460 | + return cursors; |
| 461 | + }; |
| 462 | + assertEquals(getCursors.get(), Collections.singleton(conf.getReplicatorPrefix() + "." + remoteCluster)); |
| 463 | + |
| 464 | + if (topicLevelPolicy) { |
| 465 | + admin.topics().setReplicationClusters(topicName, Collections.emptyList()); |
| 466 | + } else { |
| 467 | + admin.namespaces().setNamespaceReplicationClustersAsync(namespace, Collections.emptySet()).get(); |
| 468 | + } |
| 469 | + admin.clusters().deleteCluster(remoteCluster); |
| 470 | + // Now the cluster and its related policy has been removed but the replicator cursor still exists |
| 471 | + |
| 472 | + topic.initialize().get(3, TimeUnit.SECONDS); |
| 473 | + Awaitility.await().atMost(3, TimeUnit.SECONDS) |
| 474 | + .until(() -> !topic.getManagedLedger().getCursors().iterator().hasNext()); |
| 475 | + } |
405 | 476 | } |
0 commit comments