|
18 | 18 | */
|
19 | 19 | package org.apache.pulsar.broker.service.persistent;
|
20 | 20 |
|
| 21 | +import static org.apache.pulsar.broker.service.persistent.PersistentTopic.DEDUPLICATION_CURSOR_NAME; |
21 | 22 | import static org.testng.Assert.assertEquals;
|
22 | 23 | import static org.testng.Assert.assertFalse;
|
23 | 24 | import static org.testng.Assert.assertNotEquals;
|
24 | 25 | import static org.testng.Assert.assertNotNull;
|
25 | 26 | import static org.testng.Assert.assertNull;
|
26 | 27 | import static org.testng.Assert.assertTrue;
|
27 | 28 | import static org.testng.Assert.fail;
|
| 29 | + |
| 30 | +import java.lang.reflect.Field; |
28 | 31 | import java.util.Optional;
|
29 | 32 | import java.util.UUID;
|
30 | 33 | import java.util.concurrent.CompletableFuture;
|
|
33 | 36 | import lombok.Cleanup;
|
34 | 37 | import org.apache.bookkeeper.mledger.ManagedCursor;
|
35 | 38 | import org.apache.bookkeeper.mledger.Position;
|
| 39 | +import org.apache.bookkeeper.mledger.impl.ManagedCursorImpl; |
| 40 | +import org.apache.bookkeeper.mledger.impl.ManagedLedgerImpl; |
| 41 | +import org.apache.pulsar.broker.BrokerTestUtil; |
| 42 | +import org.apache.pulsar.broker.service.BrokerService; |
36 | 43 | import org.apache.pulsar.broker.service.Topic;
|
37 | 44 | import org.apache.pulsar.client.api.Producer;
|
38 | 45 | import org.apache.pulsar.client.api.ProducerConsumerBase;
|
39 | 46 | import org.apache.pulsar.client.api.Schema;
|
40 | 47 | import org.apache.pulsar.common.naming.TopicName;
|
| 48 | +import org.apache.pulsar.common.util.collections.ConcurrentOpenHashMap; |
41 | 49 | import org.awaitility.Awaitility;
|
| 50 | +import org.awaitility.reflect.WhiteboxImpl; |
42 | 51 | import org.testng.Assert;
|
43 | 52 | import org.testng.annotations.AfterMethod;
|
44 | 53 | import org.testng.annotations.BeforeMethod;
|
@@ -529,6 +538,101 @@ public void testDisableNamespacePolicyTakeSnapshotShouldNotThrowException() thro
|
529 | 538 | persistentTopic.checkDeduplicationSnapshot();
|
530 | 539 | }
|
531 | 540 |
|
| 541 | + @Test |
| 542 | + public void testFinishTakeSnapshotWhenTopicLoading() throws Exception { |
| 543 | + cleanup(); |
| 544 | + setup(); |
| 545 | + |
| 546 | + // Create a topic and wait deduplication is started. |
| 547 | + int brokerDeduplicationEntriesInterval = 1000; |
| 548 | + pulsar.getConfiguration().setBrokerDeduplicationEnabled(true); |
| 549 | + pulsar.getConfiguration().setBrokerDeduplicationEntriesInterval(brokerDeduplicationEntriesInterval); |
| 550 | + final String topic = BrokerTestUtil.newUniqueName("persistent://public/default/tp"); |
| 551 | + admin.topics().createNonPartitionedTopic(topic); |
| 552 | + final PersistentTopic persistentTopic1 = |
| 553 | + (PersistentTopic) pulsar.getBrokerService().getTopic(topic, false).join().get(); |
| 554 | + final ManagedLedgerImpl ml1 = (ManagedLedgerImpl) persistentTopic1.getManagedLedger(); |
| 555 | + Awaitility.await().untilAsserted(() -> { |
| 556 | + ManagedCursorImpl cursor1 = |
| 557 | + (ManagedCursorImpl) ml1.getCursors().get(PersistentTopic.DEDUPLICATION_CURSOR_NAME); |
| 558 | + assertNotNull(cursor1); |
| 559 | + }); |
| 560 | + final MessageDeduplication deduplication1 = persistentTopic1.getMessageDeduplication(); |
| 561 | + |
| 562 | + |
| 563 | + // Send 999 messages, it is less than "brokerDeduplicationEntriesInterval". |
| 564 | + // So it would not trigger takeSnapshot |
| 565 | + final Producer<String> producer = pulsarClient.newProducer(Schema.STRING) |
| 566 | + .topic(topic).enableBatching(false).create(); |
| 567 | + for (int i = 0; i < brokerDeduplicationEntriesInterval - 1; i++) { |
| 568 | + producer.send(i + ""); |
| 569 | + } |
| 570 | + producer.close(); |
| 571 | + int snapshotCounter1 = WhiteboxImpl.getInternalState(deduplication1, "snapshotCounter"); |
| 572 | + assertEquals(snapshotCounter1, brokerDeduplicationEntriesInterval - 1); |
| 573 | + |
| 574 | + |
| 575 | + // Unload and load topic, simulate topic load is timeout. |
| 576 | + // SetBrokerDeduplicationEntriesInterval to 10, therefore recoverSequenceIdsMap#takeSnapshot |
| 577 | + // would trigger and should update the snapshot position. |
| 578 | + // However, if topic close and takeSnapshot are concurrent, |
| 579 | + // it would result in takeSnapshot throw exception |
| 580 | + admin.topics().unload(topic); |
| 581 | + pulsar.getConfiguration().setBrokerDeduplicationEntriesInterval(10); |
| 582 | + |
| 583 | + // Mock message deduplication recovery speed topicLoadTimeoutSeconds |
| 584 | + pulsar.getConfiguration().setTopicLoadTimeoutSeconds(1); |
| 585 | + String mlPath = BrokerService.MANAGED_LEDGER_PATH_ZNODE + "/" + |
| 586 | + TopicName.get(topic).getPersistenceNamingEncoding() + "/" + DEDUPLICATION_CURSOR_NAME; |
| 587 | + mockZooKeeper.delay(2 * 1000, (op, path) -> { |
| 588 | + if (mlPath.equals(path)) { |
| 589 | + return true; |
| 590 | + } |
| 591 | + return false; |
| 592 | + }); |
| 593 | + |
| 594 | + Field field2 = BrokerService.class.getDeclaredField("topics"); |
| 595 | + field2.setAccessible(true); |
| 596 | + ConcurrentOpenHashMap<String, CompletableFuture<Optional<Topic>>> topics = |
| 597 | + (ConcurrentOpenHashMap<String, CompletableFuture<Optional<Topic>>>) |
| 598 | + field2.get(pulsar.getBrokerService()); |
| 599 | + |
| 600 | + try { |
| 601 | + pulsar.getBrokerService().getTopic(topic, false).join().get(); |
| 602 | + Assert.fail(); |
| 603 | + } catch (Exception e) { |
| 604 | + // topic loading should timeout. |
| 605 | + } |
| 606 | + Awaitility.await().untilAsserted(() -> { |
| 607 | + // topic loading timeout then close topic and remove from topicsMap |
| 608 | + Assert.assertFalse(topics.containsKey(topic)); |
| 609 | + }); |
| 610 | + |
| 611 | + |
| 612 | + // Load topic again, setBrokerDeduplicationEntriesInterval to 10000, |
| 613 | + // make recoverSequenceIdsMap#takeSnapshot not trigger takeSnapshot. |
| 614 | + // But actually it should not replay again in recoverSequenceIdsMap, |
| 615 | + // since previous topic loading should finish the replay process. |
| 616 | + pulsar.getConfiguration().setBrokerDeduplicationEntriesInterval(10000); |
| 617 | + pulsar.getConfiguration().setTopicLoadTimeoutSeconds(60); |
| 618 | + PersistentTopic persistentTopic2 = |
| 619 | + (PersistentTopic) pulsar.getBrokerService().getTopic(topic, false).join().get(); |
| 620 | + ManagedLedgerImpl ml2 = (ManagedLedgerImpl) persistentTopic2.getManagedLedger(); |
| 621 | + MessageDeduplication deduplication2 = persistentTopic2.getMessageDeduplication(); |
| 622 | + |
| 623 | + Awaitility.await().untilAsserted(() -> { |
| 624 | + int snapshotCounter3 = WhiteboxImpl.getInternalState(deduplication2, "snapshotCounter"); |
| 625 | + Assert.assertEquals(snapshotCounter3, 0); |
| 626 | + Assert.assertEquals(ml2.getLedgersInfo().size(), 1); |
| 627 | + }); |
| 628 | + |
| 629 | + |
| 630 | + // cleanup. |
| 631 | + admin.topics().delete(topic); |
| 632 | + cleanup(); |
| 633 | + setup(); |
| 634 | + } |
| 635 | + |
532 | 636 | private void waitCacheInit(String topicName) throws Exception {
|
533 | 637 | pulsarClient.newConsumer().topic(topicName).subscriptionName("my-sub").subscribe().close();
|
534 | 638 | TopicName topic = TopicName.get(topicName);
|
|
0 commit comments