This is an automated email from the ASF dual-hosted git repository.
technoboy pushed a commit to branch branch-3.0
in repository https://gitbox.apache.org/repos/asf/pulsar.git
The following commit(s) were added to refs/heads/branch-3.0 by this push:
new 31d28efad11 [improve][broker] do not grant permission for each
partition to reduce unnecessary zk metadata (#18222)
31d28efad11 is described below
commit 31d28efad11017420ecc5010a317d2b0bdea8013
Author: ken <[email protected]>
AuthorDate: Mon Jul 3 17:38:59 2023 +0800
[improve][broker] do not grant permission for each partition to reduce
unnecessary zk metadata (#18222)
Co-authored-by: fanjianye <[email protected]>
---
.../pulsar/broker/admin/impl/PersistentTopicsBase.java | 17 +++--------------
.../pulsar/broker/admin/PersistentTopicsTest.java | 13 -------------
.../client/api/AuthenticatedProducerConsumerTest.java | 5 -----
3 files changed, 3 insertions(+), 32 deletions(-)
diff --git
a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java
b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java
index d892c156839..ebc838756f9 100644
---
a/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java
+++
b/pulsar-broker/src/main/java/org/apache/pulsar/broker/admin/impl/PersistentTopicsBase.java
@@ -299,20 +299,9 @@ public class PersistentTopicsBase extends AdminResource {
// This operation should be reading from zookeeper and it should be
allowed without having admin privileges
validateAdminAccessForTenantAsync(namespaceName.getTenant())
.thenCompose(__ ->
validatePoliciesReadOnlyAccessAsync().thenCompose(unused1 ->
- getPartitionedTopicMetadataAsync(topicName, true, false)
- .thenCompose(metadata -> {
- int numPartitions = metadata.partitions;
- CompletableFuture<Void> future =
CompletableFuture.completedFuture(null);
- if (numPartitions > 0) {
- for (int i = 0; i < numPartitions; i++) {
- TopicName topicNamePartition =
topicName.getPartition(i);
- future = future.thenCompose(unused ->
grantPermissionsAsync(topicNamePartition, role,
- actions));
- }
- }
- return future.thenCompose(unused ->
grantPermissionsAsync(topicName, role, actions))
- .thenAccept(unused ->
asyncResponse.resume(Response.noContent().build()));
- }))).exceptionally(ex -> {
+ grantPermissionsAsync(topicName, role, actions)
+ .thenAccept(unused ->
asyncResponse.resume(Response.noContent().build()))))
+ .exceptionally(ex -> {
Throwable realCause =
FutureUtil.unwrapCompletionException(ex);
log.error("[{}] Failed to get permissions for topic {}",
clientAppId(), topicName, realCause);
resumeAsyncResponseExceptionally(asyncResponse, realCause);
diff --git
a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PersistentTopicsTest.java
b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PersistentTopicsTest.java
index 2095582fd6f..23ea5838d56 100644
---
a/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PersistentTopicsTest.java
+++
b/pulsar-broker/src/test/java/org/apache/pulsar/broker/admin/PersistentTopicsTest.java
@@ -942,19 +942,6 @@ public class PersistentTopicsTest extends
MockedPulsarServiceBaseTest {
verify(response,
timeout(5000).times(1)).resume(responseCaptor.capture());
Map<String, Set<AuthAction>> permissions = (Map<String,
Set<AuthAction>>) responseCaptor.getValue();
Assert.assertEquals(permissions.get(role), expectActions);
- TopicName topicName = TopicName.get(TopicDomain.persistent.value(),
testTenant, testNamespace,
- partitionedTopicName);
- for (int i = 0; i < numPartitions; i++) {
- TopicName partition = topicName.getPartition(i);
- response = mock(AsyncResponse.class);
- responseCaptor = ArgumentCaptor.forClass(Response.class);
- persistentTopics.getPermissionsOnTopic(response, testTenant,
testNamespace,
- partition.getEncodedLocalName());
- verify(response,
timeout(5000).times(1)).resume(responseCaptor.capture());
- Map<String, Set<AuthAction>> partitionPermissions =
- (Map<String, Set<AuthAction>>) responseCaptor.getValue();
- Assert.assertEquals(partitionPermissions.get(role), expectActions);
- }
}
@Test
diff --git
a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/AuthenticatedProducerConsumerTest.java
b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/AuthenticatedProducerConsumerTest.java
index a189940bee8..3bd8b920a30 100644
---
a/pulsar-broker/src/test/java/org/apache/pulsar/client/api/AuthenticatedProducerConsumerTest.java
+++
b/pulsar-broker/src/test/java/org/apache/pulsar/client/api/AuthenticatedProducerConsumerTest.java
@@ -397,11 +397,6 @@ public class AuthenticatedProducerConsumerTest extends
ProducerConsumerBase {
Awaitility.await().untilAsserted(() -> {
assertTrue(pulsar.getPulsarResources().getNamespaceResources().getPolicies(NamespaceName.get("p1/ns1"))
.get().auth_policies.getTopicAuthentication().containsKey(partitionedTopic));
- for (int i = 0; i < numPartitions; i++) {
-
assertTrue(pulsar.getPulsarResources().getNamespaceResources().getPolicies(NamespaceName.get("p1/ns1"))
- .get().auth_policies.getTopicAuthentication()
-
.containsKey(TopicName.get(partitionedTopic).getPartition(i).toString()));
- }
});
admin.topics().deletePartitionedTopic("persistent://p1/ns1/partitioned-topic");