jsancio commented on code in PR #20481:
URL: https://github.com/apache/kafka/pull/20481#discussion_r2452460496
##########
core/src/main/scala/kafka/server/KafkaRequestHandler.scala:
##########
@@ -192,19 +197,49 @@ class KafkaRequestHandler(
}
+/**
+ * Factory for creating KafkaRequestHandlerPool instances with shared
aggregate metrics.
+ * All pools created by the same factory share the same aggregateThreads
counter.
+ */
+class KafkaRequestHandlerPoolFactory {
+ val aggregateThreads = new AtomicInteger(0)
+ val RequestHandlerAvgIdleMetricName = "RequestHandlerAvgIdlePercent"
Review Comment:
Make this fields private with:
```scala
private[this] val aggregateThreads = new AtomicInteger(0)
private[this] val RequestHandlerAvgIdleMetricName =
"RequestHandlerAvgIdlePercent"
```
##########
core/src/test/scala/kafka/server/KafkaRequestHandlerTest.scala:
##########
@@ -698,4 +698,78 @@ class KafkaRequestHandlerTest {
// cleanup
brokerTopicStats.close()
}
+
+ @Test
+ def testRequestThreadMetrics(): Unit = {
+ val time = Time.SYSTEM
+ val metricsBroker = new RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val metricsController = new
RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val requestChannelBroker = new RequestChannel(10, time, metricsBroker)
+ val requestChannelController = new RequestChannel(10, time,
metricsController)
+ val apiHandler = mock(classOf[ApiRequestHandler])
+
+ // Create a factory for this test
+ val factory = new KafkaRequestHandlerPoolFactory()
+
+ // Create broker pool with 4 threads
+ val brokerPool = factory.createPool(
+ 0,
+ requestChannelBroker,
+ apiHandler,
+ time,
+ 4,
Review Comment:
Does this compile? Are you missing the "broker" string?
##########
core/src/test/scala/kafka/server/KafkaRequestHandlerTest.scala:
##########
@@ -698,4 +698,78 @@ class KafkaRequestHandlerTest {
// cleanup
brokerTopicStats.close()
}
+
+ @Test
+ def testRequestThreadMetrics(): Unit = {
+ val time = Time.SYSTEM
+ val metricsBroker = new RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val metricsController = new
RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val requestChannelBroker = new RequestChannel(10, time, metricsBroker)
+ val requestChannelController = new RequestChannel(10, time,
metricsController)
+ val apiHandler = mock(classOf[ApiRequestHandler])
+
+ // Create a factory for this test
+ val factory = new KafkaRequestHandlerPoolFactory()
+
+ // Create broker pool with 4 threads
+ val brokerPool = factory.createPool(
+ 0,
+ requestChannelBroker,
+ apiHandler,
+ time,
+ 4,
+ )
+
+ // Verify global counter is updated
+ assertEquals(4, factory.aggregateThreads.get, "global counter should be 4
after broker pool")
+
+ // Create controller pool with 4 threads
+ val controllerPool = factory.createPool(
+ 0,
+ requestChannelController,
+ apiHandler,
+ time,
+ 4,
+ "controller"
+ )
+
+ // Verify global counter is updated to sum of both pools
+ assertEquals(8, factory.aggregateThreads.get, "global counter should be 8
after both pools")
+
+ val aggregateMeterField =
classOf[KafkaRequestHandlerPool].getDeclaredField("aggregateIdleMeter")
+ aggregateMeterField.setAccessible(true)
+ val aggregateMeter =
aggregateMeterField.get(brokerPool).asInstanceOf[Meter]
Review Comment:
Hmm. Did you consider getting this metric from the metrics registry?
##########
core/src/test/scala/kafka/server/KafkaRequestHandlerTest.scala:
##########
@@ -698,4 +698,78 @@ class KafkaRequestHandlerTest {
// cleanup
brokerTopicStats.close()
}
+
+ @Test
+ def testRequestThreadMetrics(): Unit = {
+ val time = Time.SYSTEM
+ val metricsBroker = new RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val metricsController = new
RequestChannelMetrics(java.util.Set.of[ApiKeys])
+ val requestChannelBroker = new RequestChannel(10, time, metricsBroker)
+ val requestChannelController = new RequestChannel(10, time,
metricsController)
+ val apiHandler = mock(classOf[ApiRequestHandler])
+
+ // Create a factory for this test
+ val factory = new KafkaRequestHandlerPoolFactory()
+
+ // Create broker pool with 4 threads
+ val brokerPool = factory.createPool(
+ 0,
+ requestChannelBroker,
+ apiHandler,
+ time,
+ 4,
+ )
+
+ // Verify global counter is updated
+ assertEquals(4, factory.aggregateThreads.get, "global counter should be 4
after broker pool")
+
+ // Create controller pool with 4 threads
+ val controllerPool = factory.createPool(
+ 0,
+ requestChannelController,
+ apiHandler,
+ time,
+ 4,
+ "controller"
+ )
+
+ // Verify global counter is updated to sum of both pools
+ assertEquals(8, factory.aggregateThreads.get, "global counter should be 8
after both pools")
+
+ val aggregateMeterField =
classOf[KafkaRequestHandlerPool].getDeclaredField("aggregateIdleMeter")
+ aggregateMeterField.setAccessible(true)
+ val aggregateMeter =
aggregateMeterField.get(brokerPool).asInstanceOf[Meter]
+
+ val perPoolIdleMeterField =
classOf[KafkaRequestHandlerPool].getDeclaredField("perPoolIdleMeter")
+ perPoolIdleMeterField.setAccessible(true)
+ val brokerPerPoolIdleMeter =
perPoolIdleMeterField.get(brokerPool).asInstanceOf[Meter]
+ val controllerPerPoolIdleMeter =
perPoolIdleMeterField.get(controllerPool).asInstanceOf[Meter]
+
+ var aggregateValue = 0.0
+ var brokerPerPoolValue = 0.0
+ var controllerPerPoolValue = 0.0
+
+ aggregateValue = aggregateMeter.oneMinuteRate()
+ brokerPerPoolValue = brokerPerPoolIdleMeter.oneMinuteRate()
+ controllerPerPoolValue = controllerPerPoolIdleMeter.oneMinuteRate()
+
+ // Verify that the meter shows reasonable idle percentage
+ assertTrue(aggregateValue >= 0.0 && aggregateValue <= 1.00, s"aggregate
idle percent should be within [0,1], got $aggregateValue")
+ assertTrue(brokerPerPoolValue >= 0.0 && brokerPerPoolValue <= 1.00,
s"broker per-pool idle percent should be within [0,1], got $brokerPerPoolValue")
+ assertTrue(controllerPerPoolValue >= 0.0 && controllerPerPoolValue <=
1.00, s"controller per-pool idle percent should be within [0,1], got
$controllerPerPoolValue")
+
+ // Test pool resizing
+ // Shrink broker pool from 4 to 2 threads
+ brokerPool.resizeThreadPool(2)
+ assertEquals(2, brokerPool.threadPoolSize.get)
+ assertEquals(4, controllerPool.threadPoolSize.get)
+ assertEquals(6, factory.aggregateThreads.get)
+
+ // Expand controller pool from 4 to 6 threads
+ controllerPool.resizeThreadPool(6)
+ assertEquals(2, brokerPool.threadPoolSize.get)
+ assertEquals(6, controllerPool.threadPoolSize.get)
+ assertEquals(8, factory.aggregateThreads.get)
Review Comment:
Looks like these lines have an indentation with extra 2 spaces.
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]