|
18 | 18 | */ |
19 | 19 | package org.apache.pulsar.broker.service; |
20 | 20 |
|
| 21 | +import static org.testng.Assert.assertEquals; |
21 | 22 | import static org.testng.Assert.assertFalse; |
22 | 23 | import static org.testng.Assert.assertTrue; |
23 | 24 | import java.time.Duration; |
24 | 25 | import java.util.Arrays; |
25 | 26 | import java.util.HashSet; |
| 27 | +import java.util.List; |
26 | 28 | import java.util.Map; |
27 | 29 | import java.util.Optional; |
28 | 30 | import java.util.concurrent.CompletableFuture; |
|
33 | 35 | import org.apache.pulsar.client.api.Schema; |
34 | 36 | import org.apache.pulsar.common.naming.TopicName; |
35 | 37 | import org.apache.pulsar.common.policies.data.TopicType; |
| 38 | +import org.apache.pulsar.common.protocol.schema.StoredSchema; |
36 | 39 | import org.apache.pulsar.zookeeper.LocalBookkeeperEnsemble; |
37 | 40 | import org.apache.pulsar.zookeeper.ZookeeperServerTest; |
38 | 41 | import org.awaitility.Awaitility; |
@@ -172,36 +175,62 @@ public void testRemoveCluster() throws Exception { |
172 | 175 | // Initialize. |
173 | 176 | final String ns1 = defaultTenant + "/" + "ns_73b1a31afce34671a5ddc48fe5ad7fc8"; |
174 | 177 | final String topic = "persistent://" + ns1 + "/___tp-5dd50794-7af8-4a34-8a0b-06188052c66a"; |
| 178 | + final String topicP0 = TopicName.get(topic).getPartition(0).toString(); |
| 179 | + final String topicP1 = TopicName.get(topic).getPartition(1).toString(); |
175 | 180 | final String topicChangeEvents = "persistent://" + ns1 + "/__change_events-partition-0"; |
176 | 181 | admin1.namespaces().createNamespace(ns1); |
177 | 182 | admin1.namespaces().setNamespaceReplicationClusters(ns1, new HashSet<>(Arrays.asList(cluster1, cluster2))); |
178 | | - admin1.topics().createNonPartitionedTopic(topic); |
| 183 | + admin1.topics().createPartitionedTopic(topic, 2); |
| 184 | + Awaitility.await().untilAsserted(() -> { |
| 185 | + assertTrue(pulsar2.getPulsarResources().getNamespaceResources().getPartitionedTopicResources() |
| 186 | + .partitionedTopicExists(TopicName.get(topic))); |
| 187 | + List<CompletableFuture<StoredSchema>> schemaList11 |
| 188 | + = pulsar1.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 189 | + assertEquals(schemaList11.size(), 0); |
| 190 | + List<CompletableFuture<StoredSchema>> schemaList21 |
| 191 | + = pulsar2.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 192 | + assertEquals(schemaList21.size(), 0); |
| 193 | + }); |
179 | 194 |
|
180 | | - // Wait for loading topic up. |
| 195 | + // Wait for copying messages. |
181 | 196 | Producer<String> p = client1.newProducer(Schema.STRING).topic(topic).create(); |
| 197 | + p.send("msg-1"); |
| 198 | + p.close(); |
182 | 199 | Awaitility.await().untilAsserted(() -> { |
183 | 200 | Map<String, CompletableFuture<Optional<Topic>>> tps = pulsar1.getBrokerService().getTopics(); |
184 | | - assertTrue(tps.containsKey(topic)); |
| 201 | + assertTrue(tps.containsKey(topicP0)); |
| 202 | + assertTrue(tps.containsKey(topicP1)); |
185 | 203 | assertTrue(tps.containsKey(topicChangeEvents)); |
| 204 | + List<CompletableFuture<StoredSchema>> schemaList12 |
| 205 | + = pulsar1.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 206 | + assertEquals(schemaList12.size(), 1); |
| 207 | + List<CompletableFuture<StoredSchema>> schemaList22 |
| 208 | + = pulsar2.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 209 | + assertEquals(schemaList12.size(), 1); |
186 | 210 | }); |
187 | 211 |
|
188 | 212 | // The topics under the namespace of the cluster-1 will be deleted. |
189 | 213 | // Verify the result. |
190 | 214 | admin1.namespaces().setNamespaceReplicationClusters(ns1, new HashSet<>(Arrays.asList(cluster2))); |
191 | 215 | Awaitility.await().atMost(Duration.ofSeconds(60)).ignoreExceptions().untilAsserted(() -> { |
192 | 216 | Map<String, CompletableFuture<Optional<Topic>>> tps = pulsar1.getBrokerService().getTopics(); |
193 | | - assertFalse(tps.containsKey(topic)); |
| 217 | + assertFalse(tps.containsKey(topicP0)); |
| 218 | + assertFalse(tps.containsKey(topicP1)); |
194 | 219 | assertFalse(tps.containsKey(topicChangeEvents)); |
195 | | - assertFalse(pulsar1.getNamespaceService().checkTopicExists(TopicName.get(topic)) |
196 | | - .get(5, TimeUnit.SECONDS).isExists()); |
197 | 220 | assertFalse(pulsar1.getNamespaceService() |
198 | 221 | .checkTopicExists(TopicName.get(topicChangeEvents)) |
199 | 222 | .get(5, TimeUnit.SECONDS).isExists()); |
| 223 | + // Verify: schema will be removed in local cluster, and remote cluster will not. |
| 224 | + List<CompletableFuture<StoredSchema>> schemaList13 |
| 225 | + = pulsar1.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 226 | + assertEquals(schemaList13.size(), 0); |
| 227 | + List<CompletableFuture<StoredSchema>> schemaList23 |
| 228 | + = pulsar2.getSchemaStorage().getAll(TopicName.get(topic).getSchemaName()).get(); |
| 229 | + assertEquals(schemaList23.size(), 1); |
200 | 230 | }); |
201 | 231 |
|
202 | 232 | // cleanup. |
203 | | - p.close(); |
204 | | - admin2.topics().delete(topic); |
| 233 | + admin2.topics().deletePartitionedTopic(topic); |
205 | 234 | admin2.namespaces().deleteNamespace(ns1); |
206 | 235 | } |
207 | 236 | } |
0 commit comments