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

houston pushed a commit to branch branch_9_10
in repository https://gitbox.apache.org/repos/asf/solr.git


The following commit(s) were added to refs/heads/branch_9_10 by this push:
     new 159623783cd SOLR-17983: Ensure ParallelHttpShardHandler records submit 
failures  (#3843)
159623783cd is described below

commit 159623783cd51278869589b844ac6bd41c5279f9
Author: Mark Robert Miller <[email protected]>
AuthorDate: Fri Dec 12 17:46:47 2025 -0600

    SOLR-17983: Ensure ParallelHttpShardHandler records submit failures  (#3843)
    
    (cherry picked from commit 7e7c9cf92e5f48780c6bf6f908816f99a0b32b75)
---
 ...llel-http-shard-handler-failure-propagation.yml |   7 ++
 .../component/ParallelHttpShardHandler.java        |  35 ++++---
 .../component/ParallelHttpShardHandlerTest.java    | 110 +++++++++++++++++++++
 3 files changed, 139 insertions(+), 13 deletions(-)

diff --git 
a/changelog/unreleased/solr-17983-parallel-http-shard-handler-failure-propagation.yml
 
b/changelog/unreleased/solr-17983-parallel-http-shard-handler-failure-propagation.yml
new file mode 100644
index 00000000000..1c19e9e0bbc
--- /dev/null
+++ 
b/changelog/unreleased/solr-17983-parallel-http-shard-handler-failure-propagation.yml
@@ -0,0 +1,7 @@
+title: Ensure ParallelHttpShardHandler records submit failures so distributed 
requests don’t hang
+type: fixed
+authors:
+  - name: Mark Miller
+links:
+  - name: SOLR-17983
+    url: https://issues.apache.org/jira/browse/SOLR-17983
diff --git 
a/solr/core/src/java/org/apache/solr/handler/component/ParallelHttpShardHandler.java
 
b/solr/core/src/java/org/apache/solr/handler/component/ParallelHttpShardHandler.java
index c1d6f9c2007..a3e964cf95f 100644
--- 
a/solr/core/src/java/org/apache/solr/handler/component/ParallelHttpShardHandler.java
+++ 
b/solr/core/src/java/org/apache/solr/handler/component/ParallelHttpShardHandler.java
@@ -17,11 +17,12 @@
 package org.apache.solr.handler.component;
 
 import java.lang.invoke.MethodHandles;
+import java.util.concurrent.CancellationException;
 import java.util.concurrent.CompletableFuture;
+import java.util.concurrent.CompletionException;
 import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ConcurrentMap;
 import java.util.concurrent.ExecutorService;
-import java.util.concurrent.FutureTask;
 import net.jcip.annotations.NotThreadSafe;
 import org.apache.solr.client.solrj.impl.LBSolrClient;
 import org.apache.solr.common.SolrException;
@@ -55,7 +56,7 @@ public class ParallelHttpShardHandler extends 
HttpShardHandler {
    * requests are processed (despite the runnables created by this class still
    * waiting). Thus, we need to track that there are attempts still in flight.
    */
-  private final ConcurrentMap<ShardResponse, FutureTask<Void>> submitFutures;
+  private final ConcurrentMap<ShardResponse, CompletableFuture<Void>> 
submitFutures;
 
   public ParallelHttpShardHandler(ParallelHttpShardHandlerFactory 
httpShardHandlerFactory) {
     super(httpShardHandlerFactory);
@@ -79,22 +80,30 @@ public class ParallelHttpShardHandler extends 
HttpShardHandler {
       SimpleSolrResponse ssr,
       ShardResponse srsp,
       long startTimeNS) {
-    FutureTask<Void> futureTask =
-        new FutureTask<>(
-            () -> super.makeShardRequest(sreq, shard, params, lbReq, ssr, 
srsp, startTimeNS), null);
     CompletableFuture<Void> completableFuture =
-        CompletableFuture.runAsync(futureTask, commExecutor);
-    submitFutures.put(srsp, futureTask);
+        CompletableFuture.runAsync(
+            () -> super.makeShardRequest(sreq, shard, params, lbReq, ssr, 
srsp, startTimeNS),
+            commExecutor);
+    submitFutures.put(srsp, completableFuture);
     completableFuture.whenComplete(
         (r, t) -> {
           try {
             if (t != null) {
-              recordShardSubmitError(
-                  srsp,
-                  new SolrException(
-                      SolrException.ErrorCode.SERVER_ERROR,
-                      "Exception occurred while trying to send a request to 
shard: " + shard,
-                      t));
+              Throwable failure = t;
+              if (failure instanceof CompletionException) {
+                CompletionException completionException = 
(CompletionException) failure;
+                if (completionException.getCause() != null) {
+                  failure = completionException.getCause();
+                }
+              }
+              if (!(failure instanceof CancellationException)) {
+                recordShardSubmitError(
+                    srsp,
+                    new SolrException(
+                        SolrException.ErrorCode.SERVER_ERROR,
+                        "Exception occurred while trying to send a request to 
shard: " + shard,
+                        failure));
+              }
             }
           } finally {
             // Remove so that we keep track of in-flight submits only
diff --git 
a/solr/core/src/test/org/apache/solr/handler/component/ParallelHttpShardHandlerTest.java
 
b/solr/core/src/test/org/apache/solr/handler/component/ParallelHttpShardHandlerTest.java
new file mode 100644
index 00000000000..f98aac8e3c3
--- /dev/null
+++ 
b/solr/core/src/test/org/apache/solr/handler/component/ParallelHttpShardHandlerTest.java
@@ -0,0 +1,110 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.solr.handler.component;
+
+import java.util.Collections;
+import java.util.List;
+import java.util.concurrent.AbstractExecutorService;
+import java.util.concurrent.TimeUnit;
+import org.apache.solr.SolrTestCaseJ4;
+import org.apache.solr.client.solrj.impl.LBSolrClient;
+import org.apache.solr.client.solrj.request.QueryRequest;
+import org.apache.solr.common.SolrException;
+import org.apache.solr.common.params.ModifiableSolrParams;
+import org.junit.Test;
+
+public class ParallelHttpShardHandlerTest extends SolrTestCaseJ4 {
+
+  private static class DirectExecutorService extends AbstractExecutorService {
+    private volatile boolean shutdown;
+
+    @Override
+    public void shutdown() {
+      shutdown = true;
+    }
+
+    @Override
+    public List<Runnable> shutdownNow() {
+      shutdown = true;
+      return Collections.emptyList();
+    }
+
+    @Override
+    public boolean isShutdown() {
+      return shutdown;
+    }
+
+    @Override
+    public boolean isTerminated() {
+      return shutdown;
+    }
+
+    @Override
+    public boolean awaitTermination(long timeout, TimeUnit unit) {
+      return shutdown;
+    }
+
+    @Override
+    public void execute(Runnable command) {
+      command.run();
+    }
+  }
+
+  @Test
+  public void testSubmitFailureIsRecordedWhenSuperThrows() throws Exception {
+    ParallelHttpShardHandlerFactory factory = new 
ParallelHttpShardHandlerFactory();
+    factory.commExecutor = new DirectExecutorService();
+    ParallelHttpShardHandler handler = new ParallelHttpShardHandler(factory);
+
+    // Force super.makeShardRequest to throw before it enqueues the response 
future.
+    handler.lbClient = null;
+
+    ShardRequest shardRequest = new ShardRequest();
+    shardRequest.params = new ModifiableSolrParams();
+    shardRequest.actualShards = new String[] {"shardA"};
+
+    ShardResponse shardResponse = new ShardResponse();
+    shardResponse.setShardRequest(shardRequest);
+    shardResponse.setShard("shardA");
+
+    HttpShardHandler.SimpleSolrResponse simpleResponse = new 
HttpShardHandler.SimpleSolrResponse();
+    shardResponse.setSolrResponse(simpleResponse);
+
+    ModifiableSolrParams params = new ModifiableSolrParams();
+    QueryRequest queryRequest = new QueryRequest(params);
+    LBSolrClient.Endpoint endpoint = new 
LBSolrClient.Endpoint("http://ignored:8983/solr";);
+    LBSolrClient.Req lbReq =
+        new LBSolrClient.Req(queryRequest, 
Collections.singletonList(endpoint));
+
+    handler.makeShardRequest(
+        shardRequest, "shardA", params, lbReq, simpleResponse, shardResponse, 
System.nanoTime());
+
+    ShardResponse recorded = handler.responses.poll(1, TimeUnit.SECONDS);
+
+    assertNotNull(
+        "The asynchronous submit should record the shard failure when 
super.makeShardRequest fails",
+        recorded);
+    assertSame(
+        "The recorded shard response should be the same instance passed into 
recordShardSubmitError",
+        shardResponse,
+        recorded);
+    assertNotNull(
+        "Expected an exception to be attached to the recorded shard response",
+        recorded.getException());
+    assertTrue(recorded.getException() instanceof SolrException);
+  }
+}

Reply via email to