Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
import org.elasticsearch.action.search.SearchShardsResponse;
import org.elasticsearch.action.support.TransportActions;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.common.breaker.CircuitBreakingException;
import org.elasticsearch.common.util.concurrent.ConcurrentCollections;
import org.elasticsearch.compute.operator.DriverProfile;
import org.elasticsearch.compute.operator.FailureCollector;
Expand Down Expand Up @@ -209,8 +210,9 @@ private static Exception unwrapFailure(Exception e) {
private void trackShardLevelFailure(ShardId shardId, boolean fatal, Exception originalEx) {
final Exception e = unwrapFailure(originalEx);
final boolean isTaskCanceledException = ExceptionsHelper.unwrap(e, TaskCancelledException.class) != null;
final boolean isCircuitBreakerException = ExceptionsHelper.unwrap(e, CircuitBreakingException.class) != null;
shardFailures.compute(shardId, (k, current) -> {
boolean mergedFatal = fatal || isTaskCanceledException;
boolean mergedFatal = fatal || isTaskCanceledException || isCircuitBreakerException;
return current == null
? new ShardFailure(mergedFatal, e)
: new ShardFailure(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,8 @@
import org.elasticsearch.action.support.PlainActionFuture;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.node.DiscoveryNodeUtils;
import org.elasticsearch.common.breaker.CircuitBreaker.Durability;
import org.elasticsearch.common.breaker.CircuitBreakingException;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.util.concurrent.ConcurrentCollections;
import org.elasticsearch.common.util.concurrent.EsExecutors;
Expand Down Expand Up @@ -271,6 +273,17 @@ public void testNonFatalFailedOnAllNodes() {
assertThat(sent.size(), equalTo(2));
}

public void testDoNotRetryCircuitBreakerException() {
var targetShards = List.of(targetShard(shard1, node1, node2));
var sent = ConcurrentCollections.newQueue();
var future = sendRequests(targetShards, false, (node, shardIds, aliasFilters, listener) -> {
sent.add(new NodeRequest(node, shardIds, aliasFilters));
runWithDelay(() -> listener.onFailure(new CircuitBreakingException("cbe", randomFrom(Durability.values())), false));
});
expectThrows(CircuitBreakingException.class, equalTo("cbe"), future::actionGet);
assertThat(sent.size(), equalTo(1));
}

static DataNodeRequestSender.TargetShard targetShard(ShardId shardId, DiscoveryNode... nodes) {
return new DataNodeRequestSender.TargetShard(shardId, new ArrayList<>(Arrays.asList(nodes)), null);
}
Expand Down