2525import static org .apache .commons .lang3 .StringUtils .isNotBlank ;
2626import static org .apache .pulsar .broker .admin .impl .PersistentTopicsBase .unsafeGetPartitionedTopicMetadataAsync ;
2727import static org .apache .pulsar .broker .lookup .TopicLookupBase .lookupTopicAsync ;
28+ import static org .apache .pulsar .broker .service .ServerCnxThrottleTracker .ThrottleType ;
2829import static org .apache .pulsar .broker .service .persistent .PersistentTopic .getMigratedClusterUrl ;
2930import static org .apache .pulsar .broker .service .schema .BookkeeperSchemaStorage .ignoreUnrecoverableBKException ;
3031import static org .apache .pulsar .common .api .proto .ProtocolVersion .v5 ;
@@ -283,7 +284,8 @@ public void incrementPublishBytes(long bytes, long maxPendingBytesPerThread) {
283284 if (maxPendingBytesPerThread > 0 && pendingBytes > maxPendingBytesPerThread
284285 && !limitExceeded ) {
285286 limitExceeded = true ;
286- cnxsPerThread .get ().forEach (cnx -> cnx .throttleTracker .setPublishBufferLimiting (true ));
287+ cnxsPerThread .get ().forEach (cnx -> cnx .throttleTracker .markThrottled (
288+ ThrottleType .IOThreadMaxPendingPublishBytesExceeded ));
287289 }
288290 }
289291
@@ -293,7 +295,8 @@ public void decrementPublishBytes(long bytes, long resumeThresholdPendingBytesPe
293295 // we resume all connections sharing the same thread
294296 if (limitExceeded && pendingBytes <= resumeThresholdPendingBytesPerThread ) {
295297 limitExceeded = false ;
296- cnxsPerThread .get ().forEach (cnx -> cnx .throttleTracker .setPublishBufferLimiting (false ));
298+ cnxsPerThread .get ().forEach (cnx -> cnx .throttleTracker .unmarkThrottled (
299+ ThrottleType .IOThreadMaxPendingPublishBytesExceeded ));
297300 }
298301 }
299302 }
@@ -311,6 +314,7 @@ enum State {
311314 Start , Connected , Failed , Connecting
312315 }
313316
317+ @ Getter
314318 private final ServerCnxThrottleTracker throttleTracker ;
315319
316320 public ServerCnx (PulsarService pulsar ) {
@@ -481,12 +485,12 @@ private void checkPauseReceivingRequestsAfterResumeRateLimit(BaseCommand cmd) {
481485 log .warn ("[{}] Reached rate limitation" , this );
482486 // Stop receiving requests.
483487 pausedDueToRateLimitation = true ;
484- ctx . channel ().config (). setAutoRead ( false );
488+ getThrottleTracker ().markThrottled ( ThrottleType . ConnectionPauseReceivingCooldownRateLimit );
485489 // Resume after 1 second.
486490 ctx .channel ().eventLoop ().schedule (() -> {
487491 if (pausedDueToRateLimitation ) {
488492 log .info ("[{}] Resuming connection after rate limitation" , this );
489- ctx . channel ().config (). setAutoRead ( true );
493+ getThrottleTracker ().unmarkThrottled ( ThrottleType . ConnectionPauseReceivingCooldownRateLimit );
490494 pausedDueToRateLimitation = false ;
491495 }
492496 }, requestRateLimiter .getPeriodAtMs (), TimeUnit .MILLISECONDS );
@@ -497,7 +501,7 @@ private void checkPauseReceivingRequestsAfterResumeRateLimit(BaseCommand cmd) {
497501 public void channelWritabilityChanged (ChannelHandlerContext ctx ) throws Exception {
498502 if (pauseReceivingRequestsIfUnwritable && ctx .channel ().isWritable ()) {
499503 log .info ("[{}] is writable, turn on channel auto-read" , this );
500- ctx . channel ().config (). setAutoRead ( true );
504+ getThrottleTracker ().unmarkThrottled ( ThrottleType . ConnectionOutboundBufferFull );
501505 requestRateLimiter .timingOpen (pauseReceivingCooldownMilliSeconds , TimeUnit .MILLISECONDS );
502506 } else if (pauseReceivingRequestsIfUnwritable && !ctx .channel ().isWritable ()) {
503507 final ChannelOutboundBuffer outboundBuffer = ctx .channel ().unsafe ().outboundBuffer ();
@@ -511,7 +515,7 @@ public void channelWritabilityChanged(ChannelHandlerContext ctx) throws Exceptio
511515 PAUSE_RECEIVING_LOG .debug ("[{}] is not writable, turn off channel auto-read" , this );
512516 }
513517 }
514- ctx . channel ().config (). setAutoRead ( false );
518+ getThrottleTracker ().markThrottled ( ThrottleType . ConnectionOutboundBufferFull );
515519 }
516520 ctx .fireChannelWritabilityChanged ();
517521 }
@@ -3399,7 +3403,7 @@ public boolean isWritable() {
33993403 // or the pending publish bytes
34003404 private void increasePendingSendRequestsAndPublishBytes (int msgSize ) {
34013405 if (++pendingSendRequest == maxPendingSendRequests ) {
3402- throttleTracker .setPendingSendRequestsExceeded ( true );
3406+ throttleTracker .markThrottled ( ThrottleType . ConnectionMaxPendingPublishRequestsExceeded );
34033407 }
34043408 PendingBytesPerThreadTracker .getInstance ().incrementPublishBytes (msgSize , maxPendingBytesPerThread );
34053409 }
@@ -3424,7 +3428,7 @@ public void completedSendOperation(boolean isNonPersistentTopic, int msgSize) {
34243428 PendingBytesPerThreadTracker .getInstance ().decrementPublishBytes (msgSize , resumeThresholdPendingBytesPerThread );
34253429
34263430 if (--pendingSendRequest == resumeReadsThreshold ) {
3427- throttleTracker .setPendingSendRequestsExceeded ( false );
3431+ throttleTracker .unmarkThrottled ( ThrottleType . ConnectionMaxPendingPublishRequestsExceeded );
34283432 }
34293433
34303434 if (isNonPersistentTopic ) {
@@ -3803,22 +3807,6 @@ protected void setAuthRole(String authRole) {
38033807 this .authRole = authRole ;
38043808 }
38053809
3806- /**
3807- * {@inheritDoc}
3808- */
3809- @ Override
3810- public void incrementThrottleCount () {
3811- throttleTracker .incrementThrottleCount ();
3812- }
3813-
3814- /**
3815- * {@inheritDoc}
3816- */
3817- @ Override
3818- public void decrementThrottleCount () {
3819- throttleTracker .decrementThrottleCount ();
3820- }
3821-
38223810 @ VisibleForTesting
38233811 void setAuthState (AuthenticationState authState ) {
38243812 this .authState = authState ;
0 commit comments