Skip to content

Commit b193ec8

Browse files
committed
handle all shards unassigned
1 parent 8a0dcc6 commit b193ec8

File tree

2 files changed

+56
-13
lines changed

2 files changed

+56
-13
lines changed

x-pack/plugin/esql/src/main/java/org/elasticsearch/xpack/esql/plugin/DataNodeRequestSender.java

Lines changed: 16 additions & 13 deletions
Original file line numberDiff line numberDiff line change
@@ -195,27 +195,30 @@ private void trySendingRequestsForPendingShards(TargetShards targetShards, Compu
195195
var failure = shardFailures.get(shardId);
196196
if (failure != null && failure.fatal == false && failure.failure instanceof NoShardAvailableActionException) {
197197
pendingRetries.add(shardId);
198-
} else {
199-
shardFailures.compute(
200-
shardId,
201-
(k, v) -> new ShardFailure(
202-
true,
203-
v == null ? new NoShardAvailableActionException(shardId, "no shard copies found") : v.failure
204-
)
205-
);
206198
}
207199
}
208200
}
201+
if (pendingRetries.isEmpty() == false && remainingUnavailableShardResolutionAttempts.decrementAndGet() >= 0) {
202+
for (var entry : resolveShards(pendingRetries).entrySet()) {
203+
targetShards.getShard(entry.getKey()).remainingNodes.addAll(entry.getValue());
204+
}
205+
}
206+
for (ShardId shardId : pendingShardIds) {
207+
if (targetShards.getShard(shardId).remainingNodes.isEmpty()) {
208+
shardFailures.compute(
209+
shardId,
210+
(k, v) -> new ShardFailure(
211+
true,
212+
v == null ? new NoShardAvailableActionException(shardId, "no shard copies found") : v.failure
213+
)
214+
);
215+
}
216+
}
209217
if (reportedFailure
210218
|| (allowPartialResults == false && shardFailures.values().stream().anyMatch(shardFailure -> shardFailure.fatal))) {
211219
reportedFailure = true;
212220
reportFailures(computeListener);
213221
} else {
214-
if (pendingRetries.isEmpty() == false && remainingUnavailableShardResolutionAttempts.decrementAndGet() >= 0) {
215-
for (var entry : resolveShards(pendingRetries).entrySet()) {
216-
targetShards.getShard(entry.getKey()).remainingNodes.addAll(entry.getValue());
217-
}
218-
}
219222
for (NodeRequest request : selectNodeRequests(targetShards)) {
220223
sendOneNodeRequest(targetShards, computeListener, request);
221224
}

x-pack/plugin/esql/src/test/java/org/elasticsearch/xpack/esql/plugin/DataNodeRequestSenderTests.java

Lines changed: 40 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -501,6 +501,46 @@ public void testRetryOnlyMovedShards() {
501501
assertThat("Must retry only affected shards", resolvedShards, contains(shard2));
502502
}
503503

504+
public void testRetryUnassignedShardWithoutPartialResults() {
505+
var attempt = new AtomicInteger(0);
506+
var future = sendRequests(false, -1, List.of(targetShard(shard1, node1), targetShard(shard2, node2)), shardIds -> {
507+
attempt.incrementAndGet();
508+
return Map.of(shard1, List.of());
509+
},
510+
(node, shardIds, aliasFilters, listener) -> runWithDelay(
511+
() -> listener.onResponse(
512+
Objects.equals(shardIds, List.of(shard2))
513+
? new DataNodeComputeResponse(DriverCompletionInfo.EMPTY, Map.of())
514+
: new DataNodeComputeResponse(DriverCompletionInfo.EMPTY, Map.of(shard1, new ShardNotFoundException(shard1)))
515+
)
516+
)
517+
518+
);
519+
expectThrows(NoShardAvailableActionException.class, containsString("no such shard"), future::actionGet);
520+
}
521+
522+
public void testRetryUnassignedShardWithPartialResults() {
523+
var response = safeGet(
524+
sendRequests(
525+
true,
526+
-1,
527+
List.of(targetShard(shard1, node1), targetShard(shard2, node2)),
528+
shardIds -> Map.of(shard1, List.of()),
529+
(node, shardIds, aliasFilters, listener) -> runWithDelay(
530+
() -> listener.onResponse(
531+
Objects.equals(shardIds, List.of(shard2))
532+
? new DataNodeComputeResponse(DriverCompletionInfo.EMPTY, Map.of())
533+
: new DataNodeComputeResponse(DriverCompletionInfo.EMPTY, Map.of(shard1, new ShardNotFoundException(shard1)))
534+
)
535+
)
536+
)
537+
);
538+
assertThat(response.totalShards, equalTo(2));
539+
assertThat(response.successfulShards, equalTo(1));
540+
assertThat(response.skippedShards, equalTo(0));
541+
assertThat(response.failedShards, equalTo(1));
542+
}
543+
504544
static DataNodeRequestSender.TargetShard targetShard(ShardId shardId, DiscoveryNode... nodes) {
505545
return new DataNodeRequestSender.TargetShard(shardId, new ArrayList<>(Arrays.asList(nodes)), null);
506546
}

0 commit comments

Comments
 (0)