| 
18 | 18 |  */  | 
19 | 19 | package org.apache.pulsar.broker.service.persistent;  | 
20 | 20 | 
 
  | 
 | 21 | +import static org.mockito.Mockito.doAnswer;  | 
21 | 22 | import static org.mockito.Mockito.mock;  | 
 | 23 | +import static org.mockito.Mockito.spy;  | 
 | 24 | +import static org.testng.Assert.assertEquals;  | 
 | 25 | +import static org.testng.Assert.assertTrue;  | 
22 | 26 | import java.util.ArrayList;  | 
23 | 27 | import java.util.Arrays;  | 
24 | 28 | import java.util.Collections;  | 
25 | 29 | import java.util.LinkedList;  | 
26 | 30 | import java.util.List;  | 
 | 31 | +import java.util.concurrent.atomic.AtomicInteger;  | 
27 | 32 | import lombok.extern.slf4j.Slf4j;  | 
28 | 33 | import org.apache.bookkeeper.mledger.Entry;  | 
29 | 34 | import org.apache.bookkeeper.mledger.Position;  | 
 | 
34 | 39 | import org.apache.pulsar.broker.service.OneWayReplicatorTestBase;  | 
35 | 40 | import org.apache.pulsar.broker.service.persistent.PersistentReplicator.InFlightTask;  | 
36 | 41 | import org.apache.pulsar.client.api.MessageId;  | 
 | 42 | +import org.mockito.invocation.InvocationOnMock;  | 
 | 43 | +import org.mockito.stubbing.Answer;  | 
37 | 44 | import org.testng.Assert;  | 
38 | 45 | import org.testng.annotations.AfterClass;  | 
39 | 46 | import org.testng.annotations.BeforeClass;  | 
@@ -66,6 +73,40 @@ private void createTopics() throws Exception {  | 
66 | 73 |         admin1.topics().createSubscription(topicName, subscriptionName, MessageId.earliest);  | 
67 | 74 |     }  | 
68 | 75 | 
 
  | 
 | 76 | +    @Test  | 
 | 77 | +    public void testReplicationTaskStoppedAfterTopicClosed() throws Exception {  | 
 | 78 | +        // Close a topic, which has enabled replication.  | 
 | 79 | +        final String topicName = BrokerTestUtil.newUniqueName("persistent://" + replicatedNamespace + "/tp_");  | 
 | 80 | +        admin1.topics().createNonPartitionedTopic(topicName);  | 
 | 81 | +        waitReplicatorStarted(topicName, pulsar2);  | 
 | 82 | +        PersistentTopic topic = (PersistentTopic) pulsar1.getBrokerService().getTopic(topicName, false)  | 
 | 83 | +                .join().get();  | 
 | 84 | +        PersistentReplicator replicator = (PersistentReplicator) topic.getReplicators().get(cluster2);  | 
 | 85 | +        admin1.topics().unload(topicName);  | 
 | 86 | + | 
 | 87 | +        // Inject a task into the "inFlightTasks" to calculate how many times the method "replicator.readMoreEntries"  | 
 | 88 | +        // has been called.  | 
 | 89 | +        AtomicInteger counter = new AtomicInteger();  | 
 | 90 | +        InFlightTask injectedTask = new InFlightTask(PositionFactory.create(1, 1), 1, replicator.getReplicatorId());  | 
 | 91 | +        injectedTask.setEntries(Collections.emptyList());  | 
 | 92 | +        InFlightTask spyTask = spy(injectedTask);  | 
 | 93 | +        replicator.inFlightTasks.add(spyTask);  | 
 | 94 | +        doAnswer(new Answer() {  | 
 | 95 | +            @Override  | 
 | 96 | +            public Object answer(InvocationOnMock invocationOnMock) throws Throwable {  | 
 | 97 | +                counter.incrementAndGet();  | 
 | 98 | +                return invocationOnMock.callRealMethod();  | 
 | 99 | +            }  | 
 | 100 | +        }).when(spyTask).getReadPos();  | 
 | 101 | + | 
 | 102 | +        // Verify: there is no scheduled task to retry to read entries to replicate.  | 
 | 103 | +        // Call "readMoreEntries" to make the issue happen.  | 
 | 104 | +        replicator.readMoreEntries();  | 
 | 105 | +        Thread.sleep(PersistentTopic.MESSAGE_RATE_BACKOFF_MS * 10);  | 
 | 106 | +        assertEquals(replicator.getState(), AbstractReplicator.State.Terminated);  | 
 | 107 | +        assertTrue(counter.get() <= 1);  | 
 | 108 | +    }  | 
 | 109 | + | 
69 | 110 |     @Test  | 
70 | 111 |     public void testCreateOrRecycleInFlightTaskIntoQueue() throws Exception {  | 
71 | 112 |         log.info("Starting testCreateOrRecycleInFlightTaskIntoQueue");  | 
 | 
0 commit comments