Merge "BUG 2854 : Do not add empty read write transactions to the replicable journal"
[controller.git] / opendaylight / md-sal / sal-remoterpc-connector / src / main / java / org / opendaylight / controller / remote / rpc / RpcBroker.java
index 3354fc3da25e990279cacec849fa131e57e5b963..7ddac673c2e84827b5cb18d015319b13e587ad97 100644 (file)
 
 package org.opendaylight.controller.remote.rpc;
 
+import static akka.pattern.Patterns.ask;
 import akka.actor.ActorRef;
 import akka.actor.Props;
+import akka.dispatch.OnComplete;
 import akka.japi.Creator;
-import org.opendaylight.controller.remote.rpc.messages.ErrorResponse;
+import akka.japi.Pair;
+import com.google.common.base.Preconditions;
+import com.google.common.util.concurrent.CheckedFuture;
+import com.google.common.util.concurrent.FutureCallback;
+import com.google.common.util.concurrent.Futures;
+import com.google.common.util.concurrent.JdkFutureAdapters;
+import com.google.common.util.concurrent.ListenableFuture;
+import java.util.Arrays;
+import java.util.Collection;
+import java.util.List;
+import org.opendaylight.controller.cluster.common.actor.AbstractUntypedActor;
+import org.opendaylight.controller.cluster.datastore.node.utils.serialization.NormalizedNodeSerializer;
+import org.opendaylight.controller.md.sal.dom.api.DOMRpcException;
+import org.opendaylight.controller.md.sal.dom.api.DOMRpcResult;
+import org.opendaylight.controller.md.sal.dom.api.DOMRpcService;
+import org.opendaylight.controller.protobuff.messages.common.NormalizedNodeMessages.Node;
 import org.opendaylight.controller.remote.rpc.messages.ExecuteRpc;
-import org.opendaylight.controller.remote.rpc.messages.GetRoutedRpc;
-import org.opendaylight.controller.remote.rpc.messages.GetRoutedRpcReply;
-import org.opendaylight.controller.remote.rpc.messages.GetRpc;
-import org.opendaylight.controller.remote.rpc.messages.GetRpcReply;
-import org.opendaylight.controller.remote.rpc.messages.InvokeRoutedRpc;
 import org.opendaylight.controller.remote.rpc.messages.InvokeRpc;
 import org.opendaylight.controller.remote.rpc.messages.RpcResponse;
-import org.opendaylight.controller.sal.core.api.Broker;
-import org.opendaylight.yangtools.yang.common.RpcResult;
-import org.opendaylight.yangtools.yang.data.api.CompositeNode;
-import org.opendaylight.yangtools.yang.model.api.SchemaContext;
+import org.opendaylight.controller.remote.rpc.registry.RpcRegistry;
+import org.opendaylight.controller.remote.rpc.utils.LatestEntryRoutingLogic;
+import org.opendaylight.controller.remote.rpc.utils.RoutingLogic;
+import org.opendaylight.controller.sal.connector.api.RpcRouter;
+import org.opendaylight.yangtools.yang.common.RpcError;
+import org.opendaylight.yangtools.yang.common.RpcError.ErrorType;
+import org.opendaylight.yangtools.yang.common.RpcResultBuilder;
+import org.opendaylight.yangtools.yang.data.api.schema.NormalizedNode;
+import org.opendaylight.yangtools.yang.model.api.SchemaPath;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
-import java.util.concurrent.Future;
-
 /**
  * Actor to initiate execution of remote RPC on other nodes of the cluster.
  */
 
 public class RpcBroker extends AbstractUntypedActor {
 
-  private static final Logger LOG = LoggerFactory.getLogger(RpcBroker.class);
-  private final Broker.ProviderSession brokerSession;
-  private final ActorRef rpcRegistry;
-  private final SchemaContext schemaContext;
-
-  private RpcBroker(Broker.ProviderSession brokerSession, ActorRef rpcRegistry, SchemaContext schemaContext){
-    this.brokerSession = brokerSession;
-    this.rpcRegistry = rpcRegistry;
-    this.schemaContext = schemaContext;
-  }
-
-  public static Props props(final Broker.ProviderSession brokerSession, final ActorRef rpcRegistry, final SchemaContext schemaContext){
-    return Props.create(new Creator<RpcBroker>(){
-
-      @Override
-      public RpcBroker create() throws Exception {
-        return new RpcBroker(brokerSession, rpcRegistry, schemaContext);
-      }
-    });
-  }
-  @Override
-  protected void handleReceive(Object message) throws Exception {
-    if(message instanceof InvokeRoutedRpc) {
-      invokeRemoteRoutedRpc((InvokeRoutedRpc) message);
-    } else if(message instanceof InvokeRpc) {
-      invokeRemoteRpc((InvokeRpc) message);
-    } else if(message instanceof ExecuteRpc) {
-      executeRpc((ExecuteRpc) message);
+    private static final Logger LOG = LoggerFactory.getLogger(RpcBroker.class);
+    private final ActorRef rpcRegistry;
+    private final RemoteRpcProviderConfig config;
+    private final DOMRpcService rpcService;
+
+    private RpcBroker(final DOMRpcService rpcService, final ActorRef rpcRegistry) {
+        this.rpcService = rpcService;
+        this.rpcRegistry = rpcRegistry;
+        config = new RemoteRpcProviderConfig(getContext().system().settings().config());
     }
-  }
-
-  private void invokeRemoteRoutedRpc(InvokeRoutedRpc msg) {
-    // Look up the remote actor to execute rpc
-    LOG.debug("Looking up the remote actor for route {}", msg);
-    try {
-      RouteIdentifierImpl routeId = new RouteIdentifierImpl(null, msg.getRpc(), msg.getIdentifier());
-      GetRoutedRpc routedRpcMsg = new GetRoutedRpc(routeId);
-      GetRoutedRpcReply rpcReply = (GetRoutedRpcReply)ActorUtil.executeLocalOperation(rpcRegistry, routedRpcMsg, ActorUtil.LOCAL_ASK_DURATION, ActorUtil.LOCAL_AWAIT_DURATION);
-
-      String remoteActorPath = rpcReply.getRoutePath();
-      if(remoteActorPath == null) {
-        LOG.debug("No remote actor found for rpc execution.");
-
-        getSender().tell(new ErrorResponse(
-          new IllegalStateException("No remote actor found for rpc execution.")), self());
-      } else {
-
-        ExecuteRpc executeMsg = new ExecuteRpc(XmlUtils.inputCompositeNodeToXml(msg.getInput(), schemaContext), msg.getRpc());
-
-        Object operationRes = ActorUtil.executeRemoteOperation(this.context().actorSelection(remoteActorPath),
-            executeMsg, ActorUtil.REMOTE_ASK_DURATION, ActorUtil.REMOTE_AWAIT_DURATION);
-
-        getSender().tell(operationRes, self());
-      }
-    } catch (Exception e) {
-        LOG.error(e.toString());
-        getSender().tell(new ErrorResponse(e), self());
+
+    public static Props props(final DOMRpcService rpcService, final ActorRef rpcRegistry) {
+        Preconditions.checkNotNull(rpcRegistry, "ActorRef can not be null!");
+        Preconditions.checkNotNull(rpcService, "DOMRpcService can not be null");
+        return Props.create(new RpcBrokerCreator(rpcService, rpcRegistry));
     }
-  }
-
-  private void invokeRemoteRpc(InvokeRpc msg) {
-    // Look up the remote actor to execute rpc
-    LOG.debug("Looking up the remote actor for route {}", msg);
-    try {
-      RouteIdentifierImpl routeId = new RouteIdentifierImpl(null, msg.getRpc(), null);
-      GetRpc rpcMsg = new GetRpc(routeId);
-      GetRpcReply rpcReply = (GetRpcReply)ActorUtil.executeLocalOperation(rpcRegistry, rpcMsg, ActorUtil.LOCAL_ASK_DURATION, ActorUtil.LOCAL_AWAIT_DURATION);
-      String remoteActorPath = rpcReply.getRoutePath();
-
-      if(remoteActorPath == null) {
-        LOG.debug("No remote actor found for rpc execution.");
-
-        getSender().tell(new ErrorResponse(
-          new IllegalStateException("No remote actor found for rpc execution.")), self());
-      } else {
-        ExecuteRpc executeMsg = new ExecuteRpc(XmlUtils.inputCompositeNodeToXml(msg.getInput(), schemaContext), msg.getRpc());
-        Object operationRes = ActorUtil.executeRemoteOperation(this.context().actorSelection(remoteActorPath),
-            executeMsg, ActorUtil.REMOTE_ASK_DURATION, ActorUtil.REMOTE_AWAIT_DURATION);
-
-        getSender().tell(operationRes, self());
-      }
-    } catch (Exception e) {
-        LOG.error(e.toString());
-        getSender().tell(new ErrorResponse(e), self());
+
+    @Override
+    protected void handleReceive(final Object message) throws Exception {
+        if(message instanceof InvokeRpc) {
+            invokeRemoteRpc((InvokeRpc) message);
+        } else if(message instanceof ExecuteRpc) {
+            executeRpc((ExecuteRpc) message);
+        }
     }
-  }
-
-  private void executeRpc(ExecuteRpc msg) {
-    LOG.debug("Executing rpc for rpc {}", msg.getRpc());
-    try {
-      Future<RpcResult<CompositeNode>> rpc = brokerSession.rpc(msg.getRpc(), XmlUtils.inputXmlToCompositeNode(msg.getRpc(), msg.getInputCompositeNode(), schemaContext));
-      RpcResult<CompositeNode> rpcResult = rpc != null ? rpc.get():null;
-
-      CompositeNode result = rpcResult != null ? rpcResult.getResult() : null;
-      getSender().tell(new RpcResponse(XmlUtils.outputCompositeNodeToXml(result, schemaContext)), self());
-    } catch (Exception e) {
-      LOG.error(e.toString());
-      getSender().tell(new ErrorResponse(e), self());
+
+    private void invokeRemoteRpc(final InvokeRpc msg) {
+        if(LOG.isDebugEnabled()) {
+            LOG.debug("Looking up the remote actor for rpc {}", msg.getRpc());
+        }
+        final RpcRouter.RouteIdentifier<?,?,?> routeId = new RouteIdentifierImpl(
+                null, msg.getRpc(), msg.getIdentifier());
+        final RpcRegistry.Messages.FindRouters findMsg = new RpcRegistry.Messages.FindRouters(routeId);
+
+        final scala.concurrent.Future<Object> future = ask(rpcRegistry, findMsg, config.getAskDuration());
+
+        final ActorRef sender = getSender();
+        final ActorRef self = self();
+
+        final OnComplete<Object> onComplete = new OnComplete<Object>() {
+            @Override
+            public void onComplete(final Throwable failure, final Object reply) throws Throwable {
+                if(failure != null) {
+                    LOG.error("FindRouters failed", failure);
+                    sender.tell(new akka.actor.Status.Failure(failure), self);
+                    return;
+                }
+
+                final RpcRegistry.Messages.FindRoutersReply findReply =
+                                                (RpcRegistry.Messages.FindRoutersReply)reply;
+
+                final List<Pair<ActorRef, Long>> actorRefList = findReply.getRouterWithUpdateTime();
+
+                if(actorRefList == null || actorRefList.isEmpty()) {
+                    final String message = String.format(
+                            "No remote implementation found for rpc %s",  msg.getRpc());
+                    sender.tell(new akka.actor.Status.Failure(new RpcErrorsException(
+                            message, Arrays.asList(RpcResultBuilder.newError(ErrorType.RPC,
+                                    "operation-not-supported", message)))), self);
+                    return;
+                }
+
+                finishInvokeRpc(actorRefList, msg, sender, self);
+            }
+        };
+
+        future.onComplete(onComplete, getContext().dispatcher());
     }
-  }
 
+    protected void finishInvokeRpc(final List<Pair<ActorRef, Long>> actorRefList,
+            final InvokeRpc msg, final ActorRef sender, final ActorRef self) {
+
+        final RoutingLogic logic = new LatestEntryRoutingLogic(actorRefList);
+
+        final Node serializedNode = NormalizedNodeSerializer.serialize(msg.getInput());
+        final ExecuteRpc executeMsg = new ExecuteRpc(serializedNode, msg.getRpc());
+
+        final scala.concurrent.Future<Object> future = ask(logic.select(), executeMsg, config.getAskDuration());
+
+        final OnComplete<Object> onComplete = new OnComplete<Object>() {
+            @Override
+            public void onComplete(final Throwable failure, final Object reply) throws Throwable {
+                if(failure != null) {
+                    LOG.error("ExecuteRpc failed", failure);
+                    sender.tell(new akka.actor.Status.Failure(failure), self);
+                    return;
+                }
+
+                sender.tell(reply, self);
+            }
+        };
+
+        future.onComplete(onComplete, getContext().dispatcher());
+    }
+
+    private void executeRpc(final ExecuteRpc msg) {
+        if(LOG.isDebugEnabled()) {
+            LOG.debug("Executing rpc {}", msg.getRpc());
+        }
+        final NormalizedNode<?, ?> input = NormalizedNodeSerializer.deSerialize(msg.getInputNormalizedNode());
+        final SchemaPath schemaPath = SchemaPath.create(true, msg.getRpc());
+
+        final CheckedFuture<DOMRpcResult, DOMRpcException> future = rpcService.invokeRpc(schemaPath, input);
+
+        final ListenableFuture<DOMRpcResult> listenableFuture =
+                JdkFutureAdapters.listenInPoolThread(future);
+
+        final ActorRef sender = getSender();
+        final ActorRef self = self();
+
+        Futures.addCallback(listenableFuture, new FutureCallback<DOMRpcResult>() {
+            @Override
+            public void onSuccess(final DOMRpcResult result) {
+                if (result.getErrors() != null && ( ! result.getErrors().isEmpty())) {
+                    final String message = String.format("Execution of RPC %s failed",  msg.getRpc());
+                    Collection<RpcError> errors = result.getErrors();
+                    if(errors == null || errors.size() == 0) {
+                        errors = Arrays.asList(RpcResultBuilder.newError(ErrorType.RPC,
+                                null, message));
+                    }
+
+                    sender.tell(new akka.actor.Status.Failure(new RpcErrorsException(
+                            message, errors)), self);
+                } else {
+                    final Node serializedResultNode = NormalizedNodeSerializer.serialize(result.getResult());
+                    sender.tell(new RpcResponse(serializedResultNode), self);
+                }
+            }
+
+            @Override
+            public void onFailure(final Throwable t) {
+                LOG.error("executeRpc for {} failed: {}", msg.getRpc(), t);
+                sender.tell(new akka.actor.Status.Failure(t), self);
+            }
+        });
+    }
+
+    private static class RpcBrokerCreator implements Creator<RpcBroker> {
+        private static final long serialVersionUID = 1L;
+
+        final DOMRpcService rpcService;
+        final ActorRef rpcRegistry;
+
+        RpcBrokerCreator(final DOMRpcService rpcService, final ActorRef rpcRegistry) {
+            this.rpcService = rpcService;
+            this.rpcRegistry = rpcRegistry;
+        }
+
+        @Override
+        public RpcBroker create() throws Exception {
+            return new RpcBroker(rpcService, rpcRegistry);
+        }
+    }
 }