You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@avro.apache.org by cu...@apache.org on 2010/12/09 23:09:54 UTC

svn commit: r1044144 - in /avro/trunk: ./ lang/java/src/java/org/apache/avro/ipc/ lang/java/src/java/org/apache/avro/tool/

Author: cutting
Date: Thu Dec  9 22:09:41 2010
New Revision: 1044144

URL: http://svn.apache.org/viewvc?rev=1044144&view=rev
Log:
AVRO-707. Java: Promote SASL-based RPC.

Added:
    avro/trunk/lang/java/src/java/org/apache/avro/ipc/Ipc.java
Modified:
    avro/trunk/CHANGES.txt
    avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketServer.java
    avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketTransceiver.java
    avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcReceiveTool.java
    avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcSendTool.java

Modified: avro/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/avro/trunk/CHANGES.txt?rev=1044144&r1=1044143&r2=1044144&view=diff
==============================================================================
--- avro/trunk/CHANGES.txt (original)
+++ avro/trunk/CHANGES.txt Thu Dec  9 22:09:41 2010
@@ -43,6 +43,8 @@ Avro 1.5.0 (unreleased)
 
     AVRO-692. Java: Permit Avro 1.2 format files to be read. (cutting)
 
+    AVRO-707. Java: Promote SASL-based RPC. (cutting)
+
   BUG FIXES
 
     AVRO-675. C: Bytes and fixed setters don't update datum size.

Added: avro/trunk/lang/java/src/java/org/apache/avro/ipc/Ipc.java
URL: http://svn.apache.org/viewvc/avro/trunk/lang/java/src/java/org/apache/avro/ipc/Ipc.java?rev=1044144&view=auto
==============================================================================
--- avro/trunk/lang/java/src/java/org/apache/avro/ipc/Ipc.java (added)
+++ avro/trunk/lang/java/src/java/org/apache/avro/ipc/Ipc.java Thu Dec  9 22:09:41 2010
@@ -0,0 +1,53 @@
+/**
+ * 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.avro.ipc;
+
+import java.io.IOException;
+import java.net.InetSocketAddress;
+import java.net.URI;
+
+/** IPC utilities, including client and server factories. */
+public class Ipc {
+  private Ipc() {}                                // no public ctor
+
+  /** Create a client {@link Transceiver} connecting to the provided URI. */
+  public static Transceiver createTransceiver(URI uri) throws IOException {
+    if ("http".equals(uri.getScheme()))
+       return new HttpTransceiver(uri.toURL());
+    else if ("avro".equals(uri.getScheme()))
+      return new SaslSocketTransceiver
+        (new InetSocketAddress(uri.getHost(), uri.getPort()));
+    else
+      throw new IOException("unknown uri scheme: "+uri);
+  }
+
+  /** Create a {@link Server} listening at the named URI using the provided
+   * responder. */
+  public static Server createServer(Responder responder,
+                                    URI uri) throws IOException {
+    if ("http".equals(uri.getScheme()))
+      return new HttpServer(responder, uri.getPort());
+    else if ("avro".equals(uri.getScheme()))
+      return new SaslSocketServer
+        (responder, new InetSocketAddress(uri.getHost(), uri.getPort()));
+    else
+      throw new IOException("unknown uri scheme: "+uri);
+  }
+
+}

Modified: avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketServer.java
URL: http://svn.apache.org/viewvc/avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketServer.java?rev=1044144&r1=1044143&r2=1044144&view=diff
==============================================================================
--- avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketServer.java (original)
+++ avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketServer.java Thu Dec  9 22:09:41 2010
@@ -30,7 +30,9 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 /** A socket-based server implementation. This uses a simple, non-standard wire
- * protocol and is not intended for production services. */
+ * protocol and is not intended for production services.
+ * @deprecated use {@link SaslSocketServer} instead.
+ */
 public class SocketServer extends Thread implements Server {
   private static final Logger LOG = LoggerFactory.getLogger(SocketServer.class);
 

Modified: avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketTransceiver.java
URL: http://svn.apache.org/viewvc/avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketTransceiver.java?rev=1044144&r1=1044143&r2=1044144&view=diff
==============================================================================
--- avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketTransceiver.java (original)
+++ avro/trunk/lang/java/src/java/org/apache/avro/ipc/SocketTransceiver.java Thu Dec  9 22:09:41 2010
@@ -31,7 +31,9 @@ import org.slf4j.LoggerFactory;
 import org.apache.avro.Protocol;
 
 /** A socket-based {@link Transceiver} implementation.  This uses a simple,
- * non-standard wire protocol and is not intended for production services. */
+ * non-standard wire protocol and is not intended for production services.
+ * @deprecated use {@link SaslSocketTransceiver} instead.
+ */
 public class SocketTransceiver extends Transceiver {
   private static final Logger LOG
     = LoggerFactory.getLogger(SocketTransceiver.class);

Modified: avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcReceiveTool.java
URL: http://svn.apache.org/viewvc/avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcReceiveTool.java?rev=1044144&r1=1044143&r2=1044144&view=diff
==============================================================================
--- avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcReceiveTool.java (original)
+++ avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcReceiveTool.java Thu Dec  9 22:09:41 2010
@@ -35,7 +35,8 @@ import org.apache.avro.generic.GenericDa
 import org.apache.avro.generic.GenericResponder;
 import org.apache.avro.io.JsonEncoder;
 import org.apache.avro.ipc.AvroRemoteException;
-import org.apache.avro.ipc.HttpServer;
+import org.apache.avro.ipc.Ipc;
+import org.apache.avro.ipc.Server;
 import org.codehaus.jackson.JsonEncoding;
 import org.codehaus.jackson.JsonFactory;
 import org.codehaus.jackson.JsonGenerator;
@@ -50,7 +51,7 @@ public class RpcReceiveTool implements T
   /** Used to communicate between server thread (responder) and run() */
   private CountDownLatch latch;
   private Message expectedMessage;
-  HttpServer server;
+  Server server;
 
   @Override
   public String getName() {
@@ -59,7 +60,7 @@ public class RpcReceiveTool implements T
 
   @Override
   public String getShortDescription() {
-    return "Opens an HTTP RPC Server and listens for one message.";
+    return "Opens an RPC Server and listens for one message.";
   }
   
   private class SinkResponder extends GenericResponder {
@@ -161,7 +162,7 @@ public class RpcReceiveTool implements T
     this.out = out;
     
     latch = new CountDownLatch(1);
-    server = new HttpServer(new SinkResponder(protocol), uri.getPort());
+    server = Ipc.createServer(new SinkResponder(protocol), uri);
     server.start();
     out.println("Port: " + server.getPort());
     return 0;

Modified: avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcSendTool.java
URL: http://svn.apache.org/viewvc/avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcSendTool.java?rev=1044144&r1=1044143&r2=1044144&view=diff
==============================================================================
--- avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcSendTool.java (original)
+++ avro/trunk/lang/java/src/java/org/apache/avro/tool/RpcSendTool.java Thu Dec  9 22:09:41 2010
@@ -35,7 +35,8 @@ import org.apache.avro.generic.GenericDa
 import org.apache.avro.generic.GenericRequestor;
 import org.apache.avro.io.DatumWriter;
 import org.apache.avro.io.JsonEncoder;
-import org.apache.avro.ipc.HttpTransceiver;
+import org.apache.avro.ipc.Ipc;
+
 import org.codehaus.jackson.JsonEncoding;
 import org.codehaus.jackson.JsonFactory;
 import org.codehaus.jackson.JsonGenerator;
@@ -95,7 +96,8 @@ public class RpcSendTool implements Tool
       return 1;
     }
 
-    GenericRequestor client = makeClient(protocol, uri);
+    GenericRequestor client =
+      new GenericRequestor(protocol, Ipc.createTransceiver(uri));
     Object response = client.request(message.getName(), datum);
     dumpJson(out, message.getResponse(), response);
     return 0;
@@ -113,11 +115,4 @@ public class RpcSendTool implements Tool
     out.flush();
   }
 
-  private GenericRequestor makeClient(Protocol protocol, URI uri) 
-  throws IOException {
-    HttpTransceiver transceiver = 
-      new HttpTransceiver(uri.toURL());
-    GenericRequestor requestor = new GenericRequestor(protocol, transceiver);
-    return requestor;
-  }
 }