dsmiley commented on a change in pull request #1770:
URL: https://github.com/apache/lucene-solr/pull/1770#discussion_r474412138



##########
File path: 
solr/core/src/java/org/apache/solr/handler/component/HttpShardHandler.java
##########
@@ -156,36 +155,31 @@ public void submit(final ShardRequest sreq, final String 
shard, final Modifiable
       return;
     }
 
-    // all variables that set inside this listener must be at least volatile
-    responseCancellableMap.put(srsp, this.lbClient.asyncReq(lbReq, new 
AsyncListener<>() {
-      volatile long startTime = System.nanoTime();
-
-      @Override
-      public void onStart() {
-        if (tracer != null && span != null) {
-          tracer.inject(span.context(), Format.Builtin.HTTP_HEADERS, new 
SolrRequestCarrier(req));
-        }
-        SolrRequestInfo requestInfo = SolrRequestInfo.getRequestInfo();
-        if (requestInfo != null) 
req.setUserPrincipal(requestInfo.getReq().getUserPrincipal());
-      }
+    long startTime = System.nanoTime();
+    if (tracer != null && span != null) {

Review comment:
       I suspect that this code may be misplaced.  Formerly it was executed by 
the thread that called onStart.  Is that the same thread calling submit() here?

##########
File path: solr/solrj/src/java/org/apache/solr/client/solrj/SolrClient.java
##########
@@ -1290,6 +1295,39 @@ public SolrDocumentList getById(Collection<String> ids, 
SolrParams params) throw
     return request(request, null);
   }
 
+  /**
+   * Execute an asynchronous request against a Solr server for a given 
collection.
+   * This is only currently supported on the {@link Http2SolrClient} and 
{@link CloudHttp2SolrClient} for now.
+   * 
+   * @param request the request to execute
+   * @param collection the collection to execute the request against
+   *
+   * @return a {@link CompletableFuture} that tracks the progress of the async 
request. Supports cancelling requests via
+   * {@link CompletableFuture#cancel(boolean)}, adding callbacks/error 
handling using {@link CompletableFuture#whenComplete(BiConsumer)}
+   * and {@link CompletableFuture#exceptionally(Function)} methods, and other 
CompletableFuture functionality. Will
+   * complete exceptionally in case of either an {@link IOException} or {@link 
SolrServerException} during the request.
+   * Once completed, the CompletableFuture will contain a {@link NamedList} 
with the response from the server.
+   */
+  public CompletableFuture<NamedList<Object>> requestAsync(final 
SolrRequest<?> request, String collection) {
+    throw new UnsupportedOperationException("Async requests not supported on 
this Solr Client.");
+  }
+
+  /**
+   * Execute an asynchronous request against a Solr server using the request's 
collection parameter.
+   * This is only currently supported on the {@link Http2SolrClient} and 
{@link CloudHttp2SolrClient} for now.
+   *
+   * @param request the request to execute
+   *
+   * @return a {@link CompletableFuture} that tracks the progress of the async 
request. Supports cancelling requests via
+   * {@link CompletableFuture#cancel(boolean)}, adding callbacks/error 
handling using {@link CompletableFuture#whenComplete(BiConsumer)}
+   * and {@link CompletableFuture#exceptionally(Function)} methods, and other 
CompletableFuture functionality. Will
+   * complete exceptionally in case of either an {@link IOException} or {@link 
SolrServerException} during the request.
+   * Once completed, the CompletableFuture will contain a {@link NamedList} 
with the response from the server.
+   */
+  public CompletableFuture<NamedList<Object>> requestAsync(final 
SolrRequest<?> request) {

Review comment:
       If possible, I'd prefer that requestAsync simply not take a collection 
param.  Instead, both the request itself might have it, or the SolrClient might 
have it as a default.  I know lots of methods on SolrClient take it but I'd 
prefer to cease that trend for simplicity.
   
   RE having requestAsync on SolrClient at all -- I think throwing 
UnsupportedOperationException is acceptable.  We could implement a default 
implementation here last or in another issue.

##########
File path: 
solr/solrj/src/java/org/apache/solr/client/solrj/impl/BaseCloudSolrClient.java
##########
@@ -1167,8 +1191,38 @@ public RouteException(ErrorCode errorCode, 
NamedList<Throwable> throwables, Map<
     }
 
     LBSolrClient.Req req = new LBSolrClient.Req(request, theUrlList);
-    LBSolrClient.Rsp rsp = getLbClient().request(req);
-    return rsp.getResponse();
+    if (isAsyncRequest) {
+      CompletableFuture<LBSolrClient.Rsp> lbFuture = ((LBHttp2SolrClient) 
getLbClient()).requestAsync(req);
+      CompletableFuture<NamedList<Object>> apiFuture = new 
CompletableFuture<>();
+      lbFuture.whenComplete((result, throwable) -> {
+        if (lbFuture.isCompletedExceptionally()) {

Review comment:
       simply check if throwable is not null instead?

##########
File path: 
solr/solrj/src/java/org/apache/solr/client/solrj/impl/BaseCloudSolrClient.java
##########
@@ -476,8 +478,11 @@ public void registerDocCollectionWatcher(String 
collection, DocCollectionWatcher
     
assertZKStateProvider().zkStateReader.registerDocCollectionWatcher(collection, 
watcher);
   }
 
+  // TODO: async direct updates

Review comment:
       Since we're merely talking about using a CompletableFuture wrapper 
(right?), it seems fine to me.  This is internal too.

##########
File path: 
solr/solrj/src/java/org/apache/solr/client/solrj/impl/BaseCloudSolrClient.java
##########
@@ -1052,10 +1074,12 @@ public RouteException(ErrorCode errorCode, 
NamedList<Throwable> throwables, Map<
       }
     }
 
-    return resp;
+    return respFuture;

Review comment:
       very minor:  I'd prefer "rspFuture".  I see "response" abbreviated to 
"rsp" in general, much less to "resp".

##########
File path: 
solr/solrj/src/java/org/apache/solr/client/solrj/impl/Http2SolrClient.java
##########
@@ -394,26 +393,26 @@ public void onHeaders(Response response) {
               assert ObjectReleaseTracker.track(is);
               try {
                 NamedList<Object> body = processErrorsAndResponse(solrRequest, 
parser, response, is);
-                asyncListener.onSuccess(body);
-              } catch (RemoteSolrException e) {
-                if (SolrException.getRootCause(e) != CANCELLED_EXCEPTION) {
-                  asyncListener.onFailure(e);
-                }
-              } catch (SolrServerException e) {
-                asyncListener.onFailure(e);
+                future.complete(body);
+              } catch (RemoteSolrException | SolrServerException e) {
+                future.completeExceptionally(e);
               }
             });
           }
 
           @Override
           public void onFailure(Response response, Throwable failure) {
             super.onFailure(response, failure);
-            if (failure != CANCELLED_EXCEPTION) {
-              asyncListener.onFailure(new 
SolrServerException(failure.getMessage(), failure));
-            }
+            future.completeExceptionally(new 
SolrServerException(failure.getMessage(), failure));
           }
         });
-    return () -> req.abort(CANCELLED_EXCEPTION);
+    future.exceptionally((error) -> {
+      if (error instanceof CancellationException) {
+        req.abort(new Exception());

Review comment:
       I'm a little curious about the code being referenced here:
   * Why do an instanceof check on "error" at all; why not simply always 
req.abort(error)?
   * Why create a new Exception when we already have a suitable Throwable type 
in variable "error"?




----------------------------------------------------------------
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.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscr...@lucene.apache.org
For additional commands, e-mail: issues-h...@lucene.apache.org

Reply via email to