Skip to content

Commit

Permalink
Reduce coordinator logs (#17566)
Browse files Browse the repository at this point in the history
  • Loading branch information
kfaraz authored Dec 16, 2024
1 parent 29ab12c commit 0335bdd
Show file tree
Hide file tree
Showing 3 changed files with 10 additions and 16 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -491,10 +491,9 @@ static List<Interval> filterSkipIntervals(Interval totalInterval, List<Interval>
remainingStart = skipInterval.getEnd();
} else {
// Ignore this skipInterval
log.warn(
log.debug(
"skipInterval[%s] is not contained in remainingInterval[%s]",
skipInterval,
new Interval(remainingStart, remainingEnd)
skipInterval, new Interval(remainingStart, remainingEnd)
);
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -76,14 +76,10 @@
public class HttpLoadQueuePeon implements LoadQueuePeon
{
public static final TypeReference<List<DataSegmentChangeRequest>> REQUEST_ENTITY_TYPE_REF =
new TypeReference<List<DataSegmentChangeRequest>>()
{
};
new TypeReference<>() {};

public static final TypeReference<List<DataSegmentChangeResponse>> RESPONSE_ENTITY_TYPE_REF =
new TypeReference<List<DataSegmentChangeResponse>>()
{
};
new TypeReference<>() {};

private static final EmittingLogger log = new EmittingLogger(HttpLoadQueuePeon.class);

Expand Down Expand Up @@ -389,9 +385,10 @@ public void stop()
log.info("Stopping load queue peon for server[%s].", serverId);
stopped = true;

// Cancel all queued requests
queuedSegments.forEach(holder -> onRequestCompleted(holder, RequestStatus.CANCELLED));
log.info("Cancelled [%d] requests queued on server [%s].", queuedSegments.size(), serverId);
if (!queuedSegments.isEmpty()) {
queuedSegments.forEach(holder -> onRequestCompleted(holder, RequestStatus.CANCELLED));
log.info("Cancelled [%d] requests queued on server[%s].", queuedSegments.size(), serverId);
}

segmentsToDrop.clear();
segmentsToLoad.clear();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,7 +19,6 @@

package org.apache.druid.server.coordinator.loading;

import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.util.concurrent.Futures;
import com.google.common.util.concurrent.ListenableFuture;
Expand Down Expand Up @@ -350,9 +349,8 @@ public <Intermediate, Final> ListenableFuture<Final> go(
httpResponseHandler.handleResponse(httpResponse, null);
try {
List<DataSegmentChangeRequest> changeRequests = MAPPER.readValue(
request.getContent().array(), new TypeReference<List<DataSegmentChangeRequest>>()
{
}
request.getContent().array(),
HttpLoadQueuePeon.REQUEST_ENTITY_TYPE_REF
);

List<DataSegmentChangeResponse> statuses = new ArrayList<>(changeRequests.size());
Expand Down

0 comments on commit 0335bdd

Please sign in to comment.