This is an automated email from the ASF dual-hosted git repository.

karan pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/druid.git


The following commit(s) were added to refs/heads/master by this push:
     new 2c5c121c601 Add supervisorId as a dimension to task stat metrics 
(#18920)
2c5c121c601 is described below

commit 2c5c121c601e933eed252542e2ca2a8069fd820b
Author: Lucas Capistrant <[email protected]>
AuthorDate: Thu Jan 15 22:20:44 2026 -0600

    Add supervisorId as a dimension to task stat metrics (#18920)
    
    * Add supervisorId as a dimension to task stats
    
    'null' value for all tasks that are not of type SeekableStreamIndextask
    
    * Refactor how non-supervisor tasks are handled
    
    * update metrics
---
 docs/operations/metrics.md                               | 10 +++++-----
 .../embedded/indexing/KafkaClusterMetricsTest.java       |  3 ++-
 .../org/apache/druid/indexing/overlord/TaskQueue.java    | 16 ++++++++++++++--
 .../apache/druid/server/coordinator/stats/Dimension.java |  3 ++-
 4 files changed, 23 insertions(+), 9 deletions(-)

diff --git a/docs/operations/metrics.md b/docs/operations/metrics.md
index a1d1af43e8b..f663990244e 100644
--- a/docs/operations/metrics.md
+++ b/docs/operations/metrics.md
@@ -318,11 +318,11 @@ If the JVM does not support CPU time measurement for the 
current thread, `ingest
 |`segment/added/bytes`|Size in bytes of new segments created.| `dataSource`, 
`taskId`, `taskType`, `groupId`, `interval`, `tags`|Varies|
 |`segment/moved/bytes`|Size in bytes of segments moved/archived via the Move 
Task.| `dataSource`, `taskId`, `taskType`, `groupId`, `interval`, `tags`|Varies|
 |`segment/nuked/bytes`|Size in bytes of segments deleted via the Kill Task.| 
`dataSource`, `taskId`, `taskType`, `groupId`, `interval`, `tags`|Varies|
-|`task/success/count`|Number of successful tasks per emission period. This 
metric is available only if the `TaskCountStatsMonitor` module is included.| 
`dataSource`,`taskType`|Varies|
-|`task/failed/count`|Number of failed tasks per emission period. This metric 
is available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`|Varies|
-|`task/running/count`|Number of current running tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`|Varies|
-|`task/pending/count`|Number of current pending tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`|Varies|
-|`task/waiting/count`|Number of current waiting tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`|Varies|
+|`task/success/count`|Number of successful tasks per emission period. This 
metric is available only if the `TaskCountStatsMonitor` module is included.| 
`dataSource`,`taskType`, `supervisorId`|Varies|
+|`task/failed/count`|Number of failed tasks per emission period. This metric 
is available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`, `supervisorId`|Varies|
+|`task/running/count`|Number of current running tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`, `supervisorId`|Varies|
+|`task/pending/count`|Number of current pending tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`, `supervisorId`|Varies|
+|`task/waiting/count`|Number of current waiting tasks. This metric is 
available only if the `TaskCountStatsMonitor` module is 
included.|`dataSource`,`taskType`, `supervisorId`|Varies|
 |`taskSlot/total/count`|Number of total task slots per emission period. This 
metric is available only if the `TaskSlotCountStatsMonitor` module is 
included.| `category`|Varies|
 |`taskSlot/idle/count`|Number of idle task slots per emission period. This 
metric is available only if the `TaskSlotCountStatsMonitor` module is 
included.| `category`|Varies|
 |`taskSlot/used/count`|Number of busy task slots per emission period. This 
metric is available only if the `TaskSlotCountStatsMonitor` module is 
included.| `category`|Varies|
diff --git 
a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/KafkaClusterMetricsTest.java
 
b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/KafkaClusterMetricsTest.java
index 8200bb33556..b2b47c8e5f0 100644
--- 
a/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/KafkaClusterMetricsTest.java
+++ 
b/embedded-tests/src/test/java/org/apache/druid/testing/embedded/indexing/KafkaClusterMetricsTest.java
@@ -197,7 +197,8 @@ public class KafkaClusterMetricsTest extends 
EmbeddedClusterTestBase
     // Wait for a task to succeed
     overlord.latchableEmitter().waitForEventAggregate(
         event -> event.hasMetricName("task/success/count")
-                      .hasDimension(DruidMetrics.DATASOURCE, dataSource),
+                      .hasDimension(DruidMetrics.DATASOURCE, dataSource)
+                      .hasDimension(DruidMetrics.SUPERVISOR_ID, supervisorId),
         agg -> agg.hasSumAtLeast(1)
     );
     // Wait for some segments to be published
diff --git 
a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java
 
b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java
index 5f64e18e942..01447abaa83 100644
--- 
a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java
+++ 
b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java
@@ -52,6 +52,7 @@ import 
org.apache.druid.indexing.common.task.batch.parallel.SinglePhaseParallelI
 import org.apache.druid.indexing.overlord.config.DefaultTaskConfig;
 import org.apache.druid.indexing.overlord.config.TaskLockConfig;
 import org.apache.druid.indexing.overlord.config.TaskQueueConfig;
+import org.apache.druid.indexing.seekablestream.SeekableStreamIndexTask;
 import org.apache.druid.java.util.common.DateTimes;
 import org.apache.druid.java.util.common.ISE;
 import org.apache.druid.java.util.common.StringUtils;
@@ -1210,7 +1211,18 @@ public class TaskQueue
     if (task == null) {
       return RowKey.empty();
     }
-    return RowKey.with(Dimension.DATASOURCE, task.getDataSource())
-                 .and(Dimension.TASK_TYPE, task.getType());
+
+    String supervisorId = null;
+    if (task instanceof SeekableStreamIndexTask) {
+      supervisorId = ((SeekableStreamIndexTask<?, ?, ?>) 
task).getSupervisorId();
+    }
+
+    RowKey.Builder builder = RowKey.with(Dimension.DATASOURCE, 
task.getDataSource())
+                                   .with(Dimension.TASK_TYPE, task.getType());
+
+    if (supervisorId != null) {
+      builder.with(Dimension.SUPERVISOR_ID, supervisorId);
+    }
+    return builder.build();
   }
 }
diff --git 
a/server/src/main/java/org/apache/druid/server/coordinator/stats/Dimension.java 
b/server/src/main/java/org/apache/druid/server/coordinator/stats/Dimension.java
index 7001f7fa455..9f662f9186f 100644
--- 
a/server/src/main/java/org/apache/druid/server/coordinator/stats/Dimension.java
+++ 
b/server/src/main/java/org/apache/druid/server/coordinator/stats/Dimension.java
@@ -30,7 +30,8 @@ public enum Dimension
   DUTY("duty"),
   DUTY_GROUP("dutyGroup"),
   DESCRIPTION("description"),
-  SERVER("server");
+  SERVER("server"),
+  SUPERVISOR_ID("supervisorId");
 
   private final String reportedName;
 


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to