Skip to content

Commit

Permalink
[improve] [proxy] Not close the socket if lookup failed caused by too…
Browse files Browse the repository at this point in the history
… many requests (apache#21216)

Motivation: The Pulsar client will close the socket if it receives a `ServiceNotReady` error when doing a lookup. The Broker will respond to the client with a `TooManyRequests` error if there are too many lookup requests in progress, but the Pulsar Proxy responds to the client with a `ServiceNotReady` error in the same scenario.

Modifications: Make Pulsar Proxy respond to the client with a `TooManyRequests` error if there are too many lookup requests in progress.
  • Loading branch information
poorbarcode authored Sep 21, 2023
1 parent 66271e3 commit d6c3fa4
Show file tree
Hide file tree
Showing 2 changed files with 37 additions and 1 deletion.
Original file line number Diff line number Diff line change
Expand Up @@ -116,7 +116,7 @@ public void handleLookup(CommandLookupTopic lookup) {
log.debug("Lookup Request ID {} from {} rejected - {}.", clientRequestId, clientAddress,
throttlingErrorMessage);
}
writeAndFlush(Commands.newLookupErrorResponse(ServerError.ServiceNotReady,
writeAndFlush(Commands.newLookupErrorResponse(ServerError.TooManyRequests,
throttlingErrorMessage, clientRequestId));
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,18 +20,26 @@

import static org.mockito.Mockito.doReturn;
import static org.testng.Assert.assertTrue;
import static org.testng.Assert.fail;

import java.util.Optional;
import java.util.concurrent.Semaphore;
import java.util.concurrent.TimeUnit;

import lombok.Cleanup;

import org.apache.pulsar.broker.BrokerTestUtil;
import org.apache.pulsar.broker.auth.MockedPulsarServiceBaseTest;
import org.apache.pulsar.broker.authentication.AuthenticationService;
import org.apache.pulsar.client.api.Producer;
import org.apache.pulsar.client.api.PulsarClient;
import org.apache.pulsar.client.api.Schema;
import org.apache.pulsar.client.impl.BinaryProtoLookupService;
import org.apache.pulsar.client.impl.ClientCnx;
import org.apache.pulsar.client.impl.LookupService;
import org.apache.pulsar.client.impl.PulsarClientImpl;
import org.apache.pulsar.common.configuration.PulsarConfigurationLoader;
import org.apache.pulsar.common.naming.TopicName;
import org.apache.pulsar.metadata.impl.ZKMetadataStore;
import org.mockito.Mockito;
import org.testng.Assert;
Expand Down Expand Up @@ -112,4 +120,32 @@ public void testLookup() throws Exception {

Assert.assertEquals(LookupProxyHandler.REJECTED_PARTITIONS_METADATA_REQUESTS.get(), 5.0d);
}

@Test
public void testLookupThrottling() throws Exception {
PulsarClientImpl client = (PulsarClientImpl) PulsarClient.builder()
.serviceUrl(proxyService.getServiceUrl()).build();
String tpName = BrokerTestUtil.newUniqueName("persistent://public/default/tp");
LookupService lookupService = client.getLookup();
assertTrue(lookupService instanceof BinaryProtoLookupService);
ClientCnx lookupConnection = client.getCnxPool().getConnection(lookupService.resolveHost()).join();

// Make no permits to lookup.
Semaphore lookupSemaphore = proxyService.getLookupRequestSemaphore();
int availablePermits = lookupSemaphore.availablePermits();
lookupSemaphore.acquire(availablePermits);

// Verify will receive too many request exception, and the socket will not be closed.
try {
lookupService.getBroker(TopicName.get(tpName)).get();
fail("Expected too many request error.");
} catch (Exception ex) {
assertTrue(ex.getMessage().contains("Too many"));
}
assertTrue(lookupConnection.ctx().channel().isActive());

// cleanup.
lookupSemaphore.release(availablePermits);
client.close();
}
}

0 comments on commit d6c3fa4

Please sign in to comment.