You are viewing a plain text version of this content. The canonical link for it is here.
Posted to users@qpid.apache.org by Zane Bitter <zb...@redhat.com> on 2011/09/29 15:43:48 UTC

[PATCH] Multiplex SSL and non-SSL connections on the same port

As the IANA has ceased assigning separate TCP ports for SSL/TLS-encrypted
transports for services, it is desirable to be able to connect to a service
via either AMQP or AMQPS on a single port. AMQP-1.0 provides for a
STARTTLS-style mechanism to allow for this, however this is not supported
in AMQP-0.10.

This patch allows Qpid to detect whether a connection uses AMQP or AMQPS by
checking for an SSL/TLS Client Hello at the start of the stream. Since the
header for AMQP connections always starts with the bytes 'AMQP', it is
possible to safely detect an SSL/TLS-wrapped stream.

Detection is enabled when the SSL plugin is active and 'port' and 'ssl-port'
are set to the same value.

Signed-off-by: Zane Bitter <zb...@redhat.com>
---
 qpid/cpp/src/qpid/sys/Socket.h          |    2 
 qpid/cpp/src/qpid/sys/SslPlugin.cpp     |  145 ++++++++++++++++++++++++----
 qpid/cpp/src/qpid/sys/TCPIOPlugin.cpp   |   33 ++++++
 qpid/cpp/src/qpid/sys/ssl/SslIo.cpp     |   20 +++-
 qpid/cpp/src/qpid/sys/ssl/SslIo.h       |   16 ++-
 qpid/cpp/src/qpid/sys/ssl/SslSocket.cpp |  159 +++++++++++++++++++------------
 qpid/cpp/src/qpid/sys/ssl/SslSocket.h   |   44 ++-------
 7 files changed, 283 insertions(+), 136 deletions(-)

diff --git a/qpid/cpp/src/qpid/sys/Socket.h b/qpid/cpp/src/qpid/sys/Socket.h
index 25f1c5f..defec48 100644
--- a/qpid/cpp/src/qpid/sys/Socket.h
+++ b/qpid/cpp/src/qpid/sys/Socket.h
@@ -95,9 +95,11 @@ private:
     /** Create socket */
     void createSocket(const SocketAddress&) const;
 
+public:
     /** Construct socket with existing handle */
     Socket(IOHandlePrivate*);
 
+protected:
     mutable std::string localname;
     mutable std::string peername;
     mutable bool nonblocking;
diff --git a/qpid/cpp/src/qpid/sys/SslPlugin.cpp b/qpid/cpp/src/qpid/sys/SslPlugin.cpp
index 471a0ce..ab15785 100644
--- a/qpid/cpp/src/qpid/sys/SslPlugin.cpp
+++ b/qpid/cpp/src/qpid/sys/SslPlugin.cpp
@@ -25,6 +25,8 @@
 #include "qpid/sys/ssl/check.h"
 #include "qpid/sys/ssl/util.h"
 #include "qpid/sys/ssl/SslHandler.h"
+#include "qpid/sys/AsynchIOHandler.h"
+#include "qpid/sys/AsynchIO.h"
 #include "qpid/sys/ssl/SslIo.h"
 #include "qpid/sys/ssl/SslSocket.h"
 #include "qpid/broker/Broker.h"
@@ -37,15 +39,19 @@
 namespace qpid {
 namespace sys {
 
+using namespace qpid::sys::ssl;
+
 struct SslServerOptions : ssl::SslOptions
 {
     uint16_t port;
     bool clientAuth;
     bool nodict;
+    bool multiplex;
 
     SslServerOptions() : port(5671),
                          clientAuth(false),
-                         nodict(false)
+                         nodict(false),
+                         multiplex(false)
     {
         addOptions()
             ("ssl-port", optValue(port, "PORT"), "Port on which to listen for SSL connections")
@@ -56,15 +62,20 @@ struct SslServerOptions : ssl::SslOptions
     }
 };
 
-class SslProtocolFactory : public ProtocolFactory {
+template <class T>
+class SslProtocolFactoryTmpl : public ProtocolFactory {
+  private:
+
+    typedef SslAcceptorTmpl<T> SslAcceptor;
+
     const bool tcpNoDelay;
-    qpid::sys::ssl::SslSocket listener;
+    T listener;
     const uint16_t listeningPort;
-    std::auto_ptr<qpid::sys::ssl::SslAcceptor> acceptor;
+    std::auto_ptr<SslAcceptor> acceptor;
     bool nodict;
 
   public:
-    SslProtocolFactory(const SslServerOptions&, int backlog, bool nodelay);
+    SslProtocolFactoryTmpl(const SslServerOptions&, int backlog, bool nodelay);
     void accept(Poller::shared_ptr, ConnectionCodec::Factory*);
     void connect(Poller::shared_ptr, const std::string& host, const std::string& port,
                  ConnectionCodec::Factory*,
@@ -74,10 +85,14 @@ class SslProtocolFactory : public ProtocolFactory {
     bool supports(const std::string& capability);
 
   private:
-    void established(Poller::shared_ptr, const qpid::sys::ssl::SslSocket&, ConnectionCodec::Factory*,
+    void established(Poller::shared_ptr, const Socket&, ConnectionCodec::Factory*,
                      bool isClient);
 };
 
+typedef SslProtocolFactoryTmpl<SslSocket> SslProtocolFactory;
+typedef SslProtocolFactoryTmpl<SslMuxSocket> SslMuxProtocolFactory;
+
+
 // Static instance to initialise plugin
 static struct SslPlugin : public Plugin {
     SslServerOptions options;
@@ -86,10 +101,26 @@ static struct SslPlugin : public Plugin {
 
     ~SslPlugin() { ssl::shutdownNSS(); }
 
-    void earlyInitialize(Target&) {
+    void earlyInitialize(Target& target) {
+        broker::Broker* broker = dynamic_cast<broker::Broker*>(&target);
+        if (broker && !options.certDbPath.empty()) {
+            const broker::Broker::Options& opts = broker->getOptions();
+
+            if (opts.port == options.port && // AMQP & AMQPS ports are the same
+                opts.port != 0) {
+                // The presence of this option is used to signal to the TCP
+                // plugin not to start listening on the shared port. The actual
+                // value cannot be configured through the command line or config
+                // file (other than by setting the ports to the same value)
+                // because we are only adding it after option parsing.
+                options.multiplex = true;
+                options.addOptions()("ssl-multiplex", optValue(options.multiplex), "Allow SSL and non-SSL connections on the same port");
+            }
+        }
     }
     
     void initialize(Target& target) {
+        QPID_LOG(trace, "Initialising SSL plugin");
         broker::Broker* broker = dynamic_cast<broker::Broker*>(&target);
         // Only provide to a Broker
         if (broker) {
@@ -100,10 +131,18 @@ static struct SslPlugin : public Plugin {
                     ssl::initNSS(options, true);
                     
                     const broker::Broker::Options& opts = broker->getOptions();
-                    ProtocolFactory::shared_ptr protocol(new SslProtocolFactory(options,
-                                                                                opts.connectionBacklog,
-                                                                                opts.tcpNoDelay));
-                    QPID_LOG(notice, "Listening for SSL connections on TCP port " << protocol->getPort());
+
+                    ProtocolFactory::shared_ptr protocol(options.multiplex ?
+                        static_cast<ProtocolFactory*>(new SslMuxProtocolFactory(options,
+                                                  opts.connectionBacklog,
+                                                  opts.tcpNoDelay)) :
+                        static_cast<ProtocolFactory*>(new SslProtocolFactory(options,
+                                               opts.connectionBacklog,
+                                               opts.tcpNoDelay)));
+                    QPID_LOG(notice, "Listening for " <<
+                                     (options.multiplex ? "SSL or TCP" : "SSL") <<
+                                     " connections on TCP port " <<
+                                     protocol->getPort());
                     broker->registerProtocolFactory("ssl", protocol);
                 } catch (const std::exception& e) {
                     QPID_LOG(error, "Failed to initialise SSL plugin: " << e.what());
@@ -113,13 +152,15 @@ static struct SslPlugin : public Plugin {
     }
 } sslPlugin;
 
-SslProtocolFactory::SslProtocolFactory(const SslServerOptions& options, int backlog, bool nodelay) :
+template <class T>
+SslProtocolFactoryTmpl<T>::SslProtocolFactoryTmpl(const SslServerOptions& options, int backlog, bool nodelay) :
     tcpNoDelay(nodelay), listeningPort(listener.listen(options.port, backlog, options.certName, options.clientAuth)),
     nodict(options.nodict)
 {}
 
-void SslProtocolFactory::established(Poller::shared_ptr poller, const qpid::sys::ssl::SslSocket& s,
-                                          ConnectionCodec::Factory* f, bool isClient) {
+void SslEstablished(Poller::shared_ptr poller, const qpid::sys::SslSocket& s,
+                    ConnectionCodec::Factory* f, bool isClient,
+                    bool tcpNoDelay, bool nodict) {
     qpid::sys::ssl::SslHandler* async = new qpid::sys::ssl::SslHandler(s.getFullAddress(), f, nodict);
 
     if (tcpNoDelay) {
@@ -127,8 +168,10 @@ void SslProtocolFactory::established(Poller::shared_ptr poller, const qpid::sys:
         QPID_LOG(info, "Set TCP_NODELAY on connection to " << s.getPeerAddress());
     }
 
-    if (isClient)
+    if (isClient) {
         async->setClient();
+    }
+
     qpid::sys::ssl::SslIO* aio = new qpid::sys::ssl::SslIO(s,
                                  boost::bind(&qpid::sys::ssl::SslHandler::readbuff, async, _1, _2),
                                  boost::bind(&qpid::sys::ssl::SslHandler::eof, async, _1),
@@ -141,19 +184,64 @@ void SslProtocolFactory::established(Poller::shared_ptr poller, const qpid::sys:
     aio->start(poller);
 }
 
-uint16_t SslProtocolFactory::getPort() const {
+template <>
+void SslProtocolFactory::established(Poller::shared_ptr poller, const Socket& s,
+                                     ConnectionCodec::Factory* f, bool isClient) {
+    const SslSocket *sslSock = dynamic_cast<const SslSocket*>(&s);
+
+    SslEstablished(poller, *sslSock, f, isClient, tcpNoDelay, nodict);
+}
+
+template <class T>
+uint16_t SslProtocolFactoryTmpl<T>::getPort() const {
     return listeningPort; // Immutable no need for lock.
 }
 
-void SslProtocolFactory::accept(Poller::shared_ptr poller,
-                                     ConnectionCodec::Factory* fact) {
+template <class T>
+void SslProtocolFactoryTmpl<T>::accept(Poller::shared_ptr poller,
+                                       ConnectionCodec::Factory* fact) {
     acceptor.reset(
-        new qpid::sys::ssl::SslAcceptor(listener,
-                           boost::bind(&SslProtocolFactory::established, this, poller, _1, fact, false)));
+        new SslAcceptor(listener,
+                        boost::bind(&SslProtocolFactoryTmpl<T>::established,
+                                    this, poller, _1, fact, false)));
     acceptor->start(poller);
 }
 
-void SslProtocolFactory::connect(
+template <>
+void SslMuxProtocolFactory::established(Poller::shared_ptr poller, const Socket& s,
+                                        ConnectionCodec::Factory* f, bool isClient) {
+    const SslSocket *sslSock = dynamic_cast<const SslSocket*>(&s);
+
+    if (sslSock) {
+        SslEstablished(poller, *sslSock, f, isClient, tcpNoDelay, nodict);
+        return;
+    }
+
+    AsynchIOHandler* async = new AsynchIOHandler(s.getFullAddress(), f);
+
+    if (tcpNoDelay) {
+        s.setTcpNoDelay();
+        QPID_LOG(info, "Set TCP_NODELAY on connection to " << s.getPeerAddress());
+    }
+
+    if (isClient) {
+        async->setClient();
+    }
+    AsynchIO* aio = AsynchIO::create
+      (s,
+       boost::bind(&AsynchIOHandler::readbuff, async, _1, _2),
+       boost::bind(&AsynchIOHandler::eof, async, _1),
+       boost::bind(&AsynchIOHandler::disconnect, async, _1),
+       boost::bind(&AsynchIOHandler::closedSocket, async, _1, _2),
+       boost::bind(&AsynchIOHandler::nobuffs, async, _1),
+       boost::bind(&AsynchIOHandler::idle, async, _1));
+
+    async->init(aio, 4);
+    aio->start(poller);
+}
+
+template <class T>
+void SslProtocolFactoryTmpl<T>::connect(
     Poller::shared_ptr poller,
     const std::string& host, const std::string& port,
     ConnectionCodec::Factory* fact,
@@ -166,9 +254,9 @@ void SslProtocolFactory::connect(
     // is no longer needed.
 
     qpid::sys::ssl::SslSocket* socket = new qpid::sys::ssl::SslSocket();
-    new qpid::sys::ssl::SslConnector (*socket, poller, host, port,
-                         boost::bind(&SslProtocolFactory::established, this, poller, _1, fact, true),
-                         failed);
+    new SslConnector(*socket, poller, host, port,
+                     boost::bind(&SslProtocolFactoryTmpl<T>::established, this, poller, _1, fact, true),
+                     failed);
 }
 
 namespace
@@ -176,6 +264,7 @@ namespace
 const std::string SSL = "ssl";
 }
 
+template <>
 bool SslProtocolFactory::supports(const std::string& capability)
 {
     std::string s = capability;
@@ -183,4 +272,12 @@ bool SslProtocolFactory::supports(const std::string& capability)
     return s == SSL;
 }
 
+template <>
+bool SslMuxProtocolFactory::supports(const std::string& capability)
+{
+    std::string s = capability;
+    transform(s.begin(), s.end(), s.begin(), tolower);
+    return s == SSL || s == "tcp";
+}
+
 }} // namespace qpid::sys
diff --git a/qpid/cpp/src/qpid/sys/TCPIOPlugin.cpp b/qpid/cpp/src/qpid/sys/TCPIOPlugin.cpp
index 85d8c1d..8a99d8d 100644
--- a/qpid/cpp/src/qpid/sys/TCPIOPlugin.cpp
+++ b/qpid/cpp/src/qpid/sys/TCPIOPlugin.cpp
@@ -43,7 +43,7 @@ class AsynchIOProtocolFactory : public ProtocolFactory {
     uint16_t listeningPort;
 
   public:
-    AsynchIOProtocolFactory(const std::string& host, const std::string& port, int backlog, bool nodelay);
+    AsynchIOProtocolFactory(const std::string& host, const std::string& port, int backlog, bool nodelay, bool shouldListen);
     void accept(Poller::shared_ptr, ConnectionCodec::Factory*);
     void connect(Poller::shared_ptr, const std::string& host, const std::string& port,
                  ConnectionCodec::Factory*,
@@ -57,6 +57,20 @@ class AsynchIOProtocolFactory : public ProtocolFactory {
     void connectFailed(const Socket&, int, const std::string&, ConnectFailedCallback);
 };
 
+static bool sslMultiplexEnabled(void)
+{
+    Options o;
+    Plugin::addOptions(o);
+
+    if (o.find_nothrow("ssl-multiplex", false)) {
+        // This option is added by the SSL plugin when the SSL port
+        // is configured to be the same as the main port.
+        QPID_LOG(notice, "SSL multiplexing enabled");
+        return true;
+    }
+    return false;
+}
+
 // Static instance to initialise plugin
 static class TCPIOPlugin : public Plugin {
     void earlyInitialize(Target&) {
@@ -67,20 +81,31 @@ static class TCPIOPlugin : public Plugin {
         // Only provide to a Broker
         if (broker) {
             const broker::Broker::Options& opts = broker->getOptions();
+
+            // Check for SSL on the same port
+            bool shouldListen = !sslMultiplexEnabled();
+
             ProtocolFactory::shared_ptr protocolt(
                 new AsynchIOProtocolFactory(
                     "", boost::lexical_cast<std::string>(opts.port),
                     opts.connectionBacklog,
-                    opts.tcpNoDelay));
-            QPID_LOG(notice, "Listening on TCP/TCP6 port " << protocolt->getPort());
+                    opts.tcpNoDelay,
+                    shouldListen));
+            if (shouldListen) {
+                QPID_LOG(notice, "Listening on TCP/TCP6 port " << protocolt->getPort());
+            }
             broker->registerProtocolFactory("tcp", protocolt);
         }
     }
 } tcpPlugin;
 
-AsynchIOProtocolFactory::AsynchIOProtocolFactory(const std::string& host, const std::string& port, int backlog, bool nodelay) :
+AsynchIOProtocolFactory::AsynchIOProtocolFactory(const std::string& host, const std::string& port, int backlog, bool nodelay, bool shouldListen) :
     tcpNoDelay(nodelay)
 {
+    if (!shouldListen) {
+        return;
+    }
+
     SocketAddress sa(host, port);
 
     // We must have at least one resolved address
diff --git a/qpid/cpp/src/qpid/sys/ssl/SslIo.cpp b/qpid/cpp/src/qpid/sys/ssl/SslIo.cpp
index 734ebb4..4a59819 100644
--- a/qpid/cpp/src/qpid/sys/ssl/SslIo.cpp
+++ b/qpid/cpp/src/qpid/sys/ssl/SslIo.cpp
@@ -68,29 +68,33 @@ __thread int64_t threadMaxReadTimeNs = 2 * 1000000; // start at 2ms
  * Asynch Acceptor
  */
 
-SslAcceptor::SslAcceptor(const SslSocket& s, Callback callback) :
+template <class T>
+SslAcceptorTmpl<T>::SslAcceptorTmpl(const T& s, Callback callback) :
     acceptedCallback(callback),
-    handle(s, boost::bind(&SslAcceptor::readable, this, _1), 0, 0),
+    handle(s, boost::bind(&SslAcceptorTmpl<T>::readable, this, _1), 0, 0),
     socket(s) {
 
     s.setNonblocking();
     ignoreSigpipe();
 }
 
-SslAcceptor::~SslAcceptor() 
+template <class T>
+SslAcceptorTmpl<T>::~SslAcceptorTmpl()
 {
     handle.stopWatch();
 }
 
-void SslAcceptor::start(Poller::shared_ptr poller) {
+template <class T>
+void SslAcceptorTmpl<T>::start(Poller::shared_ptr poller) {
     handle.startWatch(poller);
 }
 
 /*
  * We keep on accepting as long as there is something to accept
  */
-void SslAcceptor::readable(DispatchHandle& h) {
-    SslSocket* s;
+template <class T>
+void SslAcceptorTmpl<T>::readable(DispatchHandle& h) {
+    Socket* s;
     do {
         errno = 0;
         // TODO: Currently we ignore the peers address, perhaps we should
@@ -110,6 +114,10 @@ void SslAcceptor::readable(DispatchHandle& h) {
     h.rewatch();
 }
 
+// Explicitly instantiate the templates we need
+template class SslAcceptorTmpl<SslSocket>;
+template class SslAcceptorTmpl<SslMuxSocket>;
+
 /*
  * Asynch Connector
  */
diff --git a/qpid/cpp/src/qpid/sys/ssl/SslIo.h b/qpid/cpp/src/qpid/sys/ssl/SslIo.h
index 8785852..c980d73 100644
--- a/qpid/cpp/src/qpid/sys/ssl/SslIo.h
+++ b/qpid/cpp/src/qpid/sys/ssl/SslIo.h
@@ -29,26 +29,30 @@
 
 namespace qpid {
 namespace sys {
+
+class Socket;
+
 namespace ssl {
-    
+
 class SslSocket;
 
 /*
  * Asynchronous ssl acceptor: accepts connections then does a callback
  * with the accepted fd
  */
-class SslAcceptor {
+template <class T>
+class SslAcceptorTmpl {
 public:
-    typedef boost::function1<void, const SslSocket&> Callback;
+    typedef boost::function1<void, const Socket&> Callback;
 
 private:
     Callback acceptedCallback;
     qpid::sys::DispatchHandle handle;
-    const SslSocket& socket;
+    const T& socket;
 
 public:
-    SslAcceptor(const SslSocket& s, Callback callback);
-    ~SslAcceptor();
+    SslAcceptorTmpl(const T& s, Callback callback);
+    ~SslAcceptorTmpl();
     void start(qpid::sys::Poller::shared_ptr poller);
 
 private:
diff --git a/qpid/cpp/src/qpid/sys/ssl/SslSocket.cpp b/qpid/cpp/src/qpid/sys/ssl/SslSocket.cpp
index f7483a2..30234bb 100644
--- a/qpid/cpp/src/qpid/sys/ssl/SslSocket.cpp
+++ b/qpid/cpp/src/qpid/sys/ssl/SslSocket.cpp
@@ -25,11 +25,13 @@
 #include "qpid/Exception.h"
 #include "qpid/sys/posix/check.h"
 #include "qpid/sys/posix/PrivatePosix.h"
+#include "qpid/log/Statement.h"
 
 #include <fcntl.h>
 #include <sys/types.h>
 #include <sys/socket.h>
 #include <sys/errno.h>
+#include <poll.h>
 #include <netinet/in.h>
 #include <netinet/tcp.h>
 #include <netdb.h>
@@ -50,36 +52,6 @@ namespace sys {
 namespace ssl {
 
 namespace {
-std::string getName(int fd, bool local, bool includeService = false)
-{
-    ::sockaddr_storage name; // big enough for any socket address
-    ::socklen_t namelen = sizeof(name);
-
-    int result = -1;
-    if (local) {
-        result = ::getsockname(fd, (::sockaddr*)&name, &namelen);
-    } else {
-        result = ::getpeername(fd, (::sockaddr*)&name, &namelen);
-    }
-
-    QPID_POSIX_CHECK(result);
-
-    char servName[NI_MAXSERV];
-    char dispName[NI_MAXHOST];
-    if (includeService) {
-        if (int rc=::getnameinfo((::sockaddr*)&name, namelen, dispName, sizeof(dispName),
-                                 servName, sizeof(servName),
-                                 NI_NUMERICHOST | NI_NUMERICSERV) != 0)
-            throw QPID_POSIX_ERROR(rc);
-        return std::string(dispName) + ":" + std::string(servName);
-
-    } else {
-        if (int rc=::getnameinfo((::sockaddr*)&name, namelen, dispName, sizeof(dispName), 0, 0, NI_NUMERICHOST) != 0)
-            throw QPID_POSIX_ERROR(rc);
-        return dispName;
-    }
-}
-
 std::string getService(int fd, bool local)
 {
     ::sockaddr_storage name; // big enough for any socket address
@@ -132,7 +104,7 @@ std::string getDomainFromSubject(std::string subject)
 
 }
 
-SslSocket::SslSocket() : IOHandle(new IOHandlePrivate()), socket(0), prototype(0)
+SslSocket::SslSocket() : socket(0), prototype(0)
 {
     impl->fd = ::socket (PF_INET, SOCK_STREAM, 0);
     if (impl->fd < 0) throw QPID_POSIX_ERROR(errno);
@@ -144,7 +116,7 @@ SslSocket::SslSocket() : IOHandle(new IOHandlePrivate()), socket(0), prototype(0
  * returned from accept. Because we use posix accept rather than
  * PR_Accept, we have to reset the handshake.
  */
-SslSocket::SslSocket(IOHandlePrivate* ioph, PRFileDesc* model) : IOHandle(ioph), socket(0), prototype(0)
+SslSocket::SslSocket(IOHandlePrivate* ioph, PRFileDesc* model) : Socket(ioph), socket(0), prototype(0)
 {
     socket = SSL_ImportFD(model, PR_ImportTCPSocket(impl->fd));
     NSS_CHECK(SSL_ResetHandshake(socket, true));
@@ -238,6 +210,7 @@ int SslSocket::listen(uint16_t port, int backlog, const std::string& certName, b
 
 SslSocket* SslSocket::accept() const
 {
+    QPID_LOG(trace, "Accepting SSL connection.");
     int afd = ::accept(impl->fd, 0, 0);
     if ( afd >= 0) {
         return new SslSocket(new IOHandlePrivate(afd), prototype);
@@ -248,36 +221,109 @@ SslSocket* SslSocket::accept() const
     }
 }
 
-int SslSocket::read(void *buf, size_t count) const
-{
-    return PR_Read(socket, buf, count);
-}
+#define SSL_STREAM_MAX_WAIT_ms 20
+#define SSL_STREAM_MAX_RETRIES 2
 
-int SslSocket::write(const void *buf, size_t count) const
-{
-    return PR_Write(socket, buf, count);
-}
+static bool isSslStream(int afd) {
+    int retries = SSL_STREAM_MAX_RETRIES;
+    unsigned char buf[5] = {};
 
-std::string SslSocket::getSockname() const
-{
-    return getName(impl->fd, true);
+    do {
+        struct pollfd fd = {afd, POLLIN, 0};
+
+        /*
+         * Note that this is blocking the accept thread, so connections that
+         * send no data can limit the rate at which we can accept new
+         * connections.
+         */
+        if (::poll(&fd, 1, SSL_STREAM_MAX_WAIT_ms) > 0) {
+            errno = 0;
+            int result = recv(afd, buf, sizeof(buf), MSG_PEEK | MSG_DONTWAIT);
+            if (result == sizeof(buf)) {
+                break;
+            }
+            if (errno && errno != EAGAIN) {
+                int err = errno;
+                ::close(afd);
+                throw QPID_POSIX_ERROR(err);
+            }
+        }
+    } while (retries-- > 0);
+
+    if (retries < 0) {
+        return false;
+    }
+
+    /*
+     * SSLv2 Client Hello format
+     * http://www.mozilla.org/projects/security/pki/nss/ssl/draft02.html
+     *
+     * Bytes 0-1: RECORD-LENGTH
+     * Byte    2: MSG-CLIENT-HELLO (1)
+     * Byte    3: CLIENT-VERSION-MSB
+     * Byte    4: CLIENT-VERSION-LSB
+     *
+     * Allowed versions:
+     * 2.0 - SSLv2
+     * 3.0 - SSLv3
+     * 3.1 - TLS 1.0
+     * 3.2 - TLS 1.1
+     * 3.3 - TLS 1.2
+     *
+     * The version sent in the Client-Hello is the latest version supported by
+     * the client. NSS may send version 3.x in an SSLv2 header for
+     * maximum compatibility.
+     */
+    bool isSSL2Handshake = buf[2] == 1 &&   // MSG-CLIENT-HELLO
+        ((buf[3] == 3 && buf[4] <= 3) ||    // SSL 3.0 & TLS 1.0-1.2 (v3.1-3.3)
+         (buf[3] == 2 && buf[4] == 0));     // SSL 2
+
+    /*
+     * SSLv3/TLS Client Hello format
+     * RFC 2246
+     *
+     * Byte    0: ContentType (handshake - 22)
+     * Bytes 1-2: ProtocolVersion {major, minor}
+     *
+     * Allowed versions:
+     * 3.0 - SSLv3
+     * 3.1 - TLS 1.0
+     * 3.2 - TLS 1.1
+     * 3.3 - TLS 1.2
+     */
+    bool isSSL3Handshake = buf[0] == 22 &&  // handshake
+        (buf[1] == 3 && buf[2] <= 3);       // SSL 3.0 & TLS 1.0-1.2 (v3.1-3.3)
+
+    return isSSL2Handshake || isSSL3Handshake;
 }
 
-std::string SslSocket::getPeername() const
+Socket* SslMuxSocket::accept() const
 {
-    return getName(impl->fd, false);
+    int afd = ::accept(impl->fd, 0, 0);
+    if (afd >= 0) {
+        QPID_LOG(trace, "Accepting connection with optional SSL wrapper.");
+        if (isSslStream(afd)) {
+            QPID_LOG(trace, "Accepted SSL connection.");
+            return new SslSocket(new IOHandlePrivate(afd), prototype);
+        } else {
+            QPID_LOG(trace, "Accepted Plaintext connection.");
+            return new Socket(new IOHandlePrivate(afd));
+        }
+    } else if (errno == EAGAIN) {
+        return 0;
+    } else {
+        throw QPID_POSIX_ERROR(errno);
+    }
 }
 
-std::string SslSocket::getPeerAddress() const
+int SslSocket::read(void *buf, size_t count) const
 {
-    if (!connectname.empty())
-        return connectname;
-    return getName(impl->fd, false, true);
+    return PR_Read(socket, buf, count);
 }
 
-std::string SslSocket::getLocalAddress() const
+int SslSocket::write(const void *buf, size_t count) const
 {
-    return getName(impl->fd, true, true);
+    return PR_Write(socket, buf, count);
 }
 
 uint16_t SslSocket::getLocalPort() const
@@ -290,17 +336,6 @@ uint16_t SslSocket::getRemotePort() const
     return atoi(getService(impl->fd, true).c_str());
 }
 
-int SslSocket::getError() const
-{
-    int       result;
-    socklen_t rSize = sizeof (result);
-
-    if (::getsockopt(impl->fd, SOL_SOCKET, SO_ERROR, &result, &rSize) < 0)
-        throw QPID_POSIX_ERROR(errno);
-
-    return result;
-}
-
 void SslSocket::setTcpNoDelay(bool nodelay) const
 {
     if (nodelay) {
diff --git a/qpid/cpp/src/qpid/sys/ssl/SslSocket.h b/qpid/cpp/src/qpid/sys/ssl/SslSocket.h
index 9938594..eabadcb 100644
--- a/qpid/cpp/src/qpid/sys/ssl/SslSocket.h
+++ b/qpid/cpp/src/qpid/sys/ssl/SslSocket.h
@@ -23,6 +23,7 @@
  */
 
 #include "qpid/sys/IOHandle.h"
+#include "qpid/sys/Socket.h"
 #include <nspr.h>
 
 #include <string>
@@ -36,7 +37,7 @@ class Duration;
 
 namespace ssl {
 
-class SslSocket : public qpid::sys::IOHandle
+class SslSocket : public qpid::sys::Socket
 {
 public:
     /** Create a socket wrapper for descriptor. */
@@ -75,45 +76,13 @@ public:
     int read(void *buf, size_t count) const;
     int write(const void *buf, size_t count) const;
 
-    /** Returns the "socket name" ie the address bound to
-     * the near end of the socket
-     */
-    std::string getSockname() const;
-
-    /** Returns the "peer name" ie the address bound to
-     * the remote end of the socket
-     */
-    std::string getPeername() const;
-
-    /**
-     * Returns an address (host and port) for the remote end of the
-     * socket
-     */
-    std::string getPeerAddress() const;
-    /**
-     * Returns an address (host and port) for the local end of the
-     * socket
-     */
-    std::string getLocalAddress() const;
-
-    /**
-     * Returns the full address of the connection: local and remote host and port.
-     */
-    std::string getFullAddress() const { return getLocalAddress()+"-"+getPeerAddress(); }
-
     uint16_t getLocalPort() const;
     uint16_t getRemotePort() const;
 
-    /**
-     * Returns the error code stored in the socket.  This may be used
-     * to determine the result of a non-blocking connect.
-     */
-    int getError() const;
-
     int getKeyLen() const;
     std::string getClientAuthId() const;
 
-private:
+protected:
     mutable std::string connectname;
     mutable PRFileDesc* socket;
     std::string certname;
@@ -126,6 +95,13 @@ private:
     mutable PRFileDesc* prototype;
 
     SslSocket(IOHandlePrivate* ioph, PRFileDesc* model);
+    friend class SslMuxSocket;
+};
+
+class SslMuxSocket : public SslSocket
+{
+public:
+    Socket* accept() const;
 };
 
 }}}


---------------------------------------------------------------------
Apache Qpid - AMQP Messaging Implementation
Project:      http://qpid.apache.org
Use/Interact: mailto:users-subscribe@qpid.apache.org