LakshSingla commented on code in PR #15025:
URL: https://github.com/apache/druid/pull/15025#discussion_r1343667297
##########
processing/src/main/java/org/apache/druid/collections/BlockingPool.java:
##########
@@ -40,8 +39,14 @@ public interface BlockingPool<T>
* Take resources from the pool, waiting if necessary until the elements of
the given number become available.
*
* @param elementNum number of resources to take
- *
* @return a list of resource holders. An empty list is returned if {@code
elementNum} resources aren't available.
*/
List<ReferenceCountingResourceHolder<T>> takeBatch(int elementNum);
+
+ /**
+ * Return the count of pending queries waiting for merge buffers
+ *
+ * @return count of pending queries
+ */
+ long getPendingQueries();
Review Comment:
Since `BlockingPool` is a generic collections class
`org.apache.druid.collections`, can you refactor the method and the doc to
something that is not specific to queries?
Of the top of my head, something like:
```suggestion
/**
* Returns the count of the requests waiting to acquire a batch of
resources.
*
* @return count of pending requests
*/
long getPendingRequests();
```
##########
server/src/main/java/org/apache/druid/server/metrics/QueryCountStatsMonitor.java:
##########
@@ -21,24 +21,30 @@
import com.google.common.collect.ImmutableMap;
import com.google.inject.Inject;
+import org.apache.druid.collections.BlockingPool;
+import org.apache.druid.guice.annotations.Merging;
import org.apache.druid.java.util.emitter.service.ServiceEmitter;
import org.apache.druid.java.util.emitter.service.ServiceMetricEvent;
import org.apache.druid.java.util.metrics.AbstractMonitor;
import org.apache.druid.java.util.metrics.KeyedDiff;
+import java.nio.ByteBuffer;
import java.util.Map;
public class QueryCountStatsMonitor extends AbstractMonitor
{
private final KeyedDiff keyedDiff = new KeyedDiff();
private final QueryCountStatsProvider statsProvider;
+ private final BlockingPool<ByteBuffer> mergeBufferPool;
@Inject
public QueryCountStatsMonitor(
- QueryCountStatsProvider statsProvider
+ QueryCountStatsProvider statsProvider,
+ @Merging BlockingPool<ByteBuffer> mergeBufferPoolIn
Review Comment:
Why is it named `mergeBufferPoolIn`?
##########
server/src/test/java/org/apache/druid/server/metrics/QueryCountStatsMonitorTest.java:
##########
@@ -85,12 +105,45 @@ public void testMonitor()
event -> (String)
event.toMap().get("metric"),
event -> (Long)
event.toMap().get("value")
));
- Assert.assertEquals(5, resultMap.size());
+ Assert.assertEquals(6, resultMap.size());
Assert.assertEquals(1L, (long) resultMap.get("query/success/count"));
Assert.assertEquals(2L, (long) resultMap.get("query/failed/count"));
Assert.assertEquals(3L, (long) resultMap.get("query/interrupted/count"));
Assert.assertEquals(4L, (long) resultMap.get("query/timeout/count"));
Assert.assertEquals(10L, (long) resultMap.get("query/count"));
+ Assert.assertEquals(0, (long) resultMap.get("mergeBuffer/pendingQueries"));
+
+ }
+
+ @Test
+ public void testMonitoringMergeBuffer()
+ {
+ executorService.submit(() -> {
+ mergeBufferPool.takeBatch(10);
+ });
+
+ int count = 0;
+ try {
+ // wait at most 10 secs for the executor thread to block
Review Comment:
10 seconds is a long time for a single test to run. Can we reduce the time
somehow? Also, let's add a failsafe timeout in the annotation, which will
ensure that we are not stuck on the test if it goes over the expected time.
```
@Test(timeout = 1_234L)
```
##########
processing/src/main/java/org/apache/druid/collections/DefaultBlockingPool.java:
##########
@@ -48,6 +49,8 @@ public class DefaultBlockingPool<T> implements BlockingPool<T>
private final Condition notEnough;
private final int maxSize;
+ private final AtomicLong pendingQueries;
Review Comment:
nit: refactor as above
--
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]
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]