This is an automated email from the ASF dual-hosted git repository. penghui pushed a commit to branch branch-2.9 in repository https://gitbox.apache.org/repos/asf/pulsar.git
commit 970c11896e48de42f16987036486be49fce132ca Author: ran <[email protected]> AuthorDate: Wed Mar 30 06:30:38 2022 +0800 # Motivation (#14895) Currently, the transaction buffer don't be closed when deleting topic. # Modification Close the transaction buffer when deleting topic. (cherry picked from commit e8d52fdb14dca62e7f0d7eb933bb7dc7fb3e916e) --- .../broker/service/persistent/PersistentTopic.java | 7 +- .../buffer/TransactionBufferCloseTest.java | 120 +++++++++++++++++++++ 2 files changed, 126 insertions(+), 1 deletion(-) diff --git a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java index 7052fc5..4f1b8cc 100644 --- a/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java +++ b/pulsar-broker/src/main/java/org/apache/pulsar/broker/service/persistent/PersistentTopic.java @@ -1155,7 +1155,8 @@ public class PersistentTopic extends AbstractTopic deleteTopicAuthenticationFuture.thenCompose( __ -> deleteSchema ? deleteSchema() : CompletableFuture.completedFuture(null)) .thenAccept(__ -> deleteTopicPolicies()) - .thenCompose(__ -> transactionBuffer.clearSnapshot()).whenComplete((v, ex) -> { + .thenCompose(__ -> transactionBufferCleanupAndClose()) + .whenComplete((v, ex) -> { if (ex != null) { log.error("[{}] Error deleting topic", topic, ex); unfenceTopicToResume(); @@ -3317,6 +3318,10 @@ public class PersistentTopic extends AbstractTopic return subscription.getPendingAckManageLedger(); } + private CompletableFuture<Void> transactionBufferCleanupAndClose() { + return transactionBuffer.clearSnapshot().thenCompose(__ -> transactionBuffer.closeAsync()); + } + public long getLastDataMessagePublishedTimestamp() { return lastDataMessagePublishedTimestamp; } diff --git a/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionBufferCloseTest.java b/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionBufferCloseTest.java new file mode 100644 index 0000000..43d31e7 --- /dev/null +++ b/pulsar-broker/src/test/java/org/apache/pulsar/broker/transaction/buffer/TransactionBufferCloseTest.java @@ -0,0 +1,120 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, + * software distributed under the License is distributed on an + * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY + * KIND, either express or implied. See the License for the + * specific language governing permissions and limitations + * under the License. + */ +package org.apache.pulsar.broker.transaction.buffer; + +import com.google.common.collect.Sets; +import lombok.extern.slf4j.Slf4j; +import org.apache.commons.lang3.RandomStringUtils; +import org.apache.pulsar.broker.transaction.TransactionTestBase; +import org.apache.pulsar.client.admin.PulsarAdminException; +import org.apache.pulsar.client.api.PulsarClientException; +import org.apache.pulsar.client.api.transaction.TransactionCoordinatorClient; +import org.apache.pulsar.client.impl.PulsarClientImpl; +import org.apache.pulsar.common.events.EventsTopicNames; +import org.apache.pulsar.common.naming.NamespaceName; +import org.apache.pulsar.common.naming.TopicDomain; +import org.apache.pulsar.common.naming.TopicName; +import org.apache.pulsar.common.policies.data.PublisherStats; +import org.apache.pulsar.common.policies.data.TenantInfoImpl; +import org.awaitility.Awaitility; +import org.testng.Assert; +import org.testng.annotations.AfterMethod; +import org.testng.annotations.BeforeMethod; +import org.testng.annotations.DataProvider; +import org.testng.annotations.Test; + +import java.util.List; +import java.util.concurrent.TimeUnit; + +/** + * Transaction buffer close test. + */ +@Slf4j +@Test(groups = "broker") +public class TransactionBufferCloseTest extends TransactionTestBase { + + @BeforeMethod + protected void setup() throws Exception { + setUpBase(1, 16, null, 0); + Awaitility.await().until(() -> ((PulsarClientImpl) pulsarClient) + .getTcClient().getState() == TransactionCoordinatorClient.State.READY); + admin.tenants().createTenant(TENANT, + new TenantInfoImpl(Sets.newHashSet("appid1"), Sets.newHashSet(CLUSTER_NAME))); + } + + @AfterMethod(alwaysRun = true) + protected void cleanup() throws Exception { + super.internalCleanup(); + } + + @DataProvider(name = "isPartition") + public Object[][] isPartition() { + return new Object[][]{ + { true }, { false } + }; + } + + @Test(timeOut = 10_000, dataProvider = "isPartition") + public void deleteTopicCloseTransactionBufferTest(boolean isPartition) throws Exception { + int expectedCount = isPartition ? 30 : 1; + TopicName topicName = createAndLoadTopic(isPartition, expectedCount); + checkSnapshotPublisherCount(topicName.getNamespace(), expectedCount); + if (isPartition) { + admin.topics().deletePartitionedTopic(topicName.getPartitionedTopicName(), true); + } else { + admin.topics().delete(topicName.getPartitionedTopicName(), true); + } + checkSnapshotPublisherCount(topicName.getNamespace(), 0); + } + + @Test(timeOut = 10_000, dataProvider = "isPartition") + public void unloadTopicCloseTransactionBufferTest(boolean isPartition) throws Exception { + int expectedCount = isPartition ? 30 : 1; + TopicName topicName = createAndLoadTopic(isPartition, expectedCount); + checkSnapshotPublisherCount(topicName.getNamespace(), expectedCount); + admin.topics().unload(topicName.getPartitionedTopicName()); + checkSnapshotPublisherCount(topicName.getNamespace(), 0); + } + + private TopicName createAndLoadTopic(boolean isPartition, int partitionCount) + throws PulsarAdminException, PulsarClientException { + String namespace = TENANT + "/ns-" + RandomStringUtils.randomAlphabetic(5); + admin.namespaces().createNamespace(namespace, 3); + String topic = namespace + "/tb-close-test-"; + if (isPartition) { + admin.topics().createPartitionedTopic(topic, partitionCount); + } + pulsarClient.newProducer() + .topic(topic) + .sendTimeout(0, TimeUnit.SECONDS) + .create() + .close(); + return TopicName.get(topic); + } + + private void checkSnapshotPublisherCount(String namespace, int expectCount) throws PulsarAdminException { + TopicName snTopicName = TopicName.get(TopicDomain.persistent.value(), NamespaceName.get(namespace), + EventsTopicNames.TRANSACTION_BUFFER_SNAPSHOT); + List<PublisherStats> publisherStatsList = + (List<PublisherStats>) admin.topics() + .getStats(snTopicName.getPartitionedTopicName()).getPublishers(); + Assert.assertEquals(publisherStatsList.size(), expectCount); + } + +}
