kfaraz commented on code in PR #18060:
URL: https://github.com/apache/druid/pull/18060#discussion_r2138972442
##########
indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java:
##########
@@ -443,8 +443,14 @@ private void startPendingTaskOnRunner(TaskEntry entry,
ListenableFuture<TaskStat
e.getMessage()
);
}
- TaskStatus taskStatus = TaskStatus.failure(task.getId(),
errorMessage);
- notifyStatus(entry, taskStatus, taskStatus.getErrorMsg());
+ try {
+ TaskStatus taskStatus = TaskStatus.failure(task.getId(),
errorMessage);
+ notifyStatus(entry, taskStatus, taskStatus.getErrorMsg());
+ emitTaskCompletionLogsAndMetrics(task, taskStatus);
+ }
+ catch (Exception e2) {
Review Comment:
Can we avoid the try-catch here? Under what conditions can these lines throw
an exception?
##########
indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java:
##########
@@ -697,6 +703,10 @@ private void notifyStatus(final TaskEntry entry, final
TaskStatus taskStatus, St
if (!taskStatus.isComplete()) {
// Nothing to do for incomplete statuses.
return;
+ } else if (entry.isComplete) {
+ // A callback() or shutdown() beat us to updating the status and has
already cleaned up this task
+ log.info("Received already-complete task[%s] with status [%s],
ignoring", task.getId(), taskStatus);
Review Comment:
We should add an `else` block here that emits a completion log.
##########
indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java:
##########
@@ -697,6 +703,10 @@ private void notifyStatus(final TaskEntry entry, final
TaskStatus taskStatus, St
if (!taskStatus.isComplete()) {
// Nothing to do for incomplete statuses.
return;
+ } else if (entry.isComplete) {
+ // A callback() or shutdown() beat us to updating the status and has
already cleaned up this task
+ log.info("Received already-complete task[%s] with status [%s],
ignoring", task.getId(), taskStatus);
Review Comment:
Nit:
```suggestion
log.info("Ignoring notification with status[%s] for already completed
task[%s].", taskStatus, task.getId());
```
##########
indexing-service/src/main/java/org/apache/druid/indexing/overlord/TaskQueue.java:
##########
@@ -1052,21 +1044,43 @@ public Map<String, Task>
getActiveTasksForDatasource(String datasource)
);
}
+ private void emitTaskCompletionLogsAndMetrics(final Task task, final
TaskStatus status)
+ {
+ if (status.isComplete()) {
+ final ServiceMetricEvent.Builder metricBuilder =
ServiceMetricEvent.builder();
+ IndexTaskUtils.setTaskDimensions(metricBuilder, task);
+ IndexTaskUtils.setTaskStatusDimensions(metricBuilder, status);
+
+ emitter.emit(metricBuilder.setMetric("task/run/time",
status.getDuration()));
+
+ log.info(
+ "Completed task[%s] with status[%s] in [%d]ms.",
+ task.getId(), status.getStatusCode(), status.getDuration()
+ );
Review Comment:
We should probably move this log line to the suggested `else` block in
`notifyStatus`, so that we have a log in case of `shutdown` as well.
--
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]