You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by yc...@apache.org on 2022/11/16 00:07:03 UTC

[cassandra] branch trunk updated: Adding endpoint verification option to client_encryption_options

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

ycai pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 36e16ee3c9 Adding endpoint verification option to client_encryption_options
36e16ee3c9 is described below

commit 36e16ee3c911c710129fcf3a69595038c3dbd385
Author: Jyothsna Konisa <jk...@apple.com>
AuthorDate: Mon Nov 14 14:16:07 2022 -0800

    Adding endpoint verification option to client_encryption_options
    
    patch by Jyothsna Konisa; reviewed by Jon Meredith, Yifan Cai for CASSANDRA-18034
---
 CHANGES.txt                                        |   1 +
 conf/cassandra.yaml                                |   1 +
 .../org/apache/cassandra/net/SocketFactory.java    |   2 +-
 .../cassandra/transport/PipelineConfigurator.java  |   8 +-
 .../apache/cassandra/transport/SimpleClient.java   |   4 +-
 test/conf/cassandra_ssl_test.truststore            | Bin 3240 -> 5295 bytes
 .../cassandra_ssl_test_endpoint_verify.keystore    | Bin 0 -> 2087 bytes
 .../test/NativeTransportEncryptionOptionsTest.java | 101 +++++++++++++++++++++
 8 files changed, 113 insertions(+), 4 deletions(-)

diff --git a/CHANGES.txt b/CHANGES.txt
index 0ea3c65c46..0c94126de7 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 4.2
+ * Adding endpoint verification option to client_encryption_options (CASSANDRA-18034)
  * Replace 'wcwidth.py' with pypi module (CASSANDRA-17287)
  * Add nodetool forcecompact to remove tombstoned or ttl'd data ignoring GC grace for given table and partition keys (CASSANDRA-17711)
  * Offer IF (NOT) EXISTS in cqlsh completion for CREATE TYPE, DROP TYPE, CREATE ROLE and DROP ROLE (CASSANDRA-16640)
diff --git a/conf/cassandra.yaml b/conf/cassandra.yaml
index fbfa468c00..ff074bddd7 100644
--- a/conf/cassandra.yaml
+++ b/conf/cassandra.yaml
@@ -1394,6 +1394,7 @@ client_encryption_options:
   keystore_password: cassandra
   # Verify client certificates
   require_client_auth: false
+  # require_endpoint_verification: false
   # Set trustore and truststore_password if require_client_auth is true
   # truststore: conf/.truststore
   # truststore_password: cassandra
diff --git a/src/java/org/apache/cassandra/net/SocketFactory.java b/src/java/org/apache/cassandra/net/SocketFactory.java
index 33fff6b7a0..b135ed5107 100644
--- a/src/java/org/apache/cassandra/net/SocketFactory.java
+++ b/src/java/org/apache/cassandra/net/SocketFactory.java
@@ -215,7 +215,7 @@ public final class SocketFactory
      * Creates a new {@link SslHandler} from provided SslContext.
      * @param peer enables endpoint verification for remote address when not null
      */
-    static SslHandler newSslHandler(Channel channel, SslContext sslContext, @Nullable InetSocketAddress peer)
+    public static SslHandler newSslHandler(Channel channel, SslContext sslContext, @Nullable InetSocketAddress peer)
     {
         if (peer == null)
             return sslContext.newHandler(channel.alloc());
diff --git a/src/java/org/apache/cassandra/transport/PipelineConfigurator.java b/src/java/org/apache/cassandra/transport/PipelineConfigurator.java
index 81ff13605e..bf48eea334 100644
--- a/src/java/org/apache/cassandra/transport/PipelineConfigurator.java
+++ b/src/java/org/apache/cassandra/transport/PipelineConfigurator.java
@@ -48,6 +48,8 @@ import org.apache.cassandra.security.ISslContextFactory;
 import org.apache.cassandra.security.SSLFactory;
 import org.apache.cassandra.transport.messages.StartupMessage;
 
+import static org.apache.cassandra.net.SocketFactory.newSslHandler;
+
 /**
  * Takes care of intializing a Netty Channel and Pipeline for client protocol connections.
  * The pipeline is first set up with some common handlers for connection limiting, dropping
@@ -181,7 +183,8 @@ public class PipelineConfigurator
                             {
                                 // Connection uses SSL/TLS, replace the detection handler with a SslHandler and so use
                                 // encryption.
-                                SslHandler sslHandler = sslContext.newHandler(channel.alloc());
+                                InetSocketAddress peer = encryptionOptions.require_endpoint_verification ? (InetSocketAddress) channel.remoteAddress() : null;
+                                SslHandler sslHandler = newSslHandler(channel, sslContext, peer);
                                 channelHandlerContext.pipeline().replace(SSL_HANDLER, SSL_HANDLER, sslHandler);
                             }
                             else
@@ -199,7 +202,8 @@ public class PipelineConfigurator
                     SslContext sslContext = SSLFactory.getOrCreateSslContext(encryptionOptions,
                                                                              encryptionOptions.require_client_auth,
                                                                              ISslContextFactory.SocketType.SERVER);
-                    channel.pipeline().addFirst(SSL_HANDLER, sslContext.newHandler(channel.alloc()));
+                    InetSocketAddress peer = encryptionOptions.require_endpoint_verification ? (InetSocketAddress) channel.remoteAddress() : null;
+                    channel.pipeline().addFirst(SSL_HANDLER, newSslHandler(channel, sslContext, peer));
                 };
             default:
                 throw new IllegalStateException("Unrecognized TLS encryption policy: " + this.tlsEncryptionPolicy);
diff --git a/src/java/org/apache/cassandra/transport/SimpleClient.java b/src/java/org/apache/cassandra/transport/SimpleClient.java
index 43bb8addee..2b57d104e4 100644
--- a/src/java/org/apache/cassandra/transport/SimpleClient.java
+++ b/src/java/org/apache/cassandra/transport/SimpleClient.java
@@ -52,6 +52,7 @@ import org.apache.cassandra.security.SSLFactory;
 import org.apache.cassandra.transport.messages.*;
 import org.apache.cassandra.utils.concurrent.UncheckedInterruptedException;
 
+import static org.apache.cassandra.net.SocketFactory.newSslHandler;
 import static org.apache.cassandra.transport.CQLMessageHandler.envelopeSize;
 import static org.apache.cassandra.transport.Flusher.MAX_FRAMED_PAYLOAD_SIZE;
 import static org.apache.cassandra.utils.concurrent.NonBlockingRateLimiter.NO_OP_LIMITER;
@@ -624,7 +625,8 @@ public class SimpleClient implements Closeable
             super.initChannel(channel);
             SslContext sslContext = SSLFactory.getOrCreateSslContext(encryptionOptions, encryptionOptions.require_client_auth,
                                                                      ISslContextFactory.SocketType.CLIENT);
-            channel.pipeline().addFirst("ssl", sslContext.newHandler(channel.alloc()));
+            InetSocketAddress peer = encryptionOptions.require_endpoint_verification ? new InetSocketAddress(host, port) : null;
+            channel.pipeline().addFirst("ssl", newSslHandler(channel, sslContext, peer));
         }
     }
 
diff --git a/test/conf/cassandra_ssl_test.truststore b/test/conf/cassandra_ssl_test.truststore
index 5ba9a9977c..ab01af30cd 100644
Binary files a/test/conf/cassandra_ssl_test.truststore and b/test/conf/cassandra_ssl_test.truststore differ
diff --git a/test/conf/cassandra_ssl_test_endpoint_verify.keystore b/test/conf/cassandra_ssl_test_endpoint_verify.keystore
new file mode 100644
index 0000000000..951385b263
Binary files /dev/null and b/test/conf/cassandra_ssl_test_endpoint_verify.keystore differ
diff --git a/test/distributed/org/apache/cassandra/distributed/test/NativeTransportEncryptionOptionsTest.java b/test/distributed/org/apache/cassandra/distributed/test/NativeTransportEncryptionOptionsTest.java
index c5a810ca25..5f2caaf695 100644
--- a/test/distributed/org/apache/cassandra/distributed/test/NativeTransportEncryptionOptionsTest.java
+++ b/test/distributed/org/apache/cassandra/distributed/test/NativeTransportEncryptionOptionsTest.java
@@ -18,18 +18,33 @@
 
 package org.apache.cassandra.distributed.test;
 
+import java.io.FileInputStream;
+import java.io.InputStream;
 import java.net.InetAddress;
+import java.security.KeyStore;
 import java.util.Collections;
 
+import javax.net.ssl.KeyManagerFactory;
+import javax.net.ssl.TrustManagerFactory;
+
 import com.google.common.collect.ImmutableMap;
 import org.junit.Assert;
+import org.junit.Rule;
 import org.junit.Test;
+import org.junit.rules.ExpectedException;
 
+import com.datastax.driver.core.SSLOptions;
+import com.datastax.driver.core.exceptions.NoHostAvailableException;
+import com.datastax.shaded.netty.handler.ssl.SslContext;
+import com.datastax.shaded.netty.handler.ssl.SslContextBuilder;
 import org.apache.cassandra.distributed.Cluster;
 import org.apache.cassandra.distributed.api.Feature;
 
 public class NativeTransportEncryptionOptionsTest extends AbstractEncryptionOptionsImpl
 {
+    @Rule
+    public ExpectedException expectedException = ExpectedException.none();
+
     @Test
     public void nodeWillNotStartWithBadKeystore() throws Throwable
     {
@@ -219,4 +234,90 @@ public class NativeTransportEncryptionOptionsTest extends AbstractEncryptionOpti
             assertCannotStartDueToConfigurationException(cluster);
         }
     }
+
+    @Test
+    public void testEndpointVerificationDisabledIpNotInSAN() throws Throwable
+    {
+        // When required_endpoint_verification is set to false, client certificate Ip/hostname should be validated
+        // The certificate in cassandra_ssl_test_outbound.keystore does not have IP/hostname embeded, so when
+        // require_endpoint_verification is false, the connection should be established
+       testEndpointVerification(false, true);
+    }
+
+    @Test
+    public void testEndpointVerificationEnabledIpNotInSAN() throws Throwable
+    {
+        // When required_endpoint_verification is set to true, client certificate Ip/hostname should be validated
+        // The certificate in cassandra_ssl_test_outbound.keystore does not have IP/hostname emebeded, so when
+        // require_endpoint_verification is true, the connection should not be established
+        testEndpointVerification(true, false);
+    }
+
+    @Test
+    public void testEndpointVerificationEnabledWithIPInSan() throws Throwable
+    {
+        // When required_endpoint_verification is set to true, client certificate Ip/hostname should be validated
+        // The certificate in cassandra_ssl_test_outbound.keystore have IP/hostname emebeded, so when
+        // require_endpoint_verification is true, the connection should be established
+        testEndpointVerification(true, true);
+    }
+
+    private void testEndpointVerification(boolean requireEndpointVerification, boolean ipInSAN) throws Throwable
+    {
+        try (Cluster cluster = builder().withNodes(1).withConfig(c -> {
+            c.with(Feature.NATIVE_PROTOCOL);
+            c.set("client_encryption_options",
+                  ImmutableMap.builder().putAll(validKeystore)
+                              .put("enabled", true)
+                              .put("require_client_auth", true)
+                              .put("require_endpoint_verification", requireEndpointVerification)
+                              .build());
+        }).start())
+        {
+            InetAddress address = cluster.get(1).config().broadcastAddress().getAddress();
+            SslContextBuilder sslContextBuilder = SslContextBuilder.forClient();
+            if (ipInSAN)
+                sslContextBuilder.keyManager(createKeyManagerFactory("test/conf/cassandra_ssl_test_endpoint_verify.keystore", "cassandra"));
+            else
+                sslContextBuilder.keyManager(createKeyManagerFactory("test/conf/cassandra_ssl_test_outbound.keystore", "cassandra"));
+
+            SslContext sslContext = sslContextBuilder.trustManager(createTrustManagerFactory("test/conf/cassandra_ssl_test.truststore", "cassandra"))
+                                                     .build();
+            final SSLOptions sslOptions = socketChannel -> sslContext.newHandler(socketChannel.alloc());
+            com.datastax.driver.core.Cluster driverCluster = com.datastax.driver.core.Cluster.builder()
+                                                                                             .addContactPoint(address.getHostAddress())
+                                                                                             .withSSL(sslOptions)
+                                                                                             .build();
+
+            if (!ipInSAN)
+            {
+                expectedException.expect(NoHostAvailableException.class);
+            }
+
+            driverCluster.connect();
+        }
+    }
+
+    private KeyManagerFactory createKeyManagerFactory(final String keyStorePath,
+                                                     final String keyStorePassword) throws Exception
+    {
+        final InputStream stream = new FileInputStream(keyStorePath);
+        final KeyStore ks = KeyStore.getInstance("JKS");
+        ks.load(stream, keyStorePassword.toCharArray());
+        final KeyManagerFactory kmf = KeyManagerFactory.getInstance(KeyManagerFactory.getDefaultAlgorithm());
+        kmf.init(ks, keyStorePassword.toCharArray());
+        return kmf;
+    }
+
+    private TrustManagerFactory createTrustManagerFactory(final String trustStorePath,
+                                                          final String trustStorePassword) throws Exception
+    {
+        final InputStream stream = new FileInputStream(trustStorePath);
+        final KeyStore ts = KeyStore.getInstance("JKS");
+        ts.load(stream, trustStorePassword.toCharArray());
+        final TrustManagerFactory tmf = TrustManagerFactory.getInstance(TrustManagerFactory.getDefaultAlgorithm());
+        tmf.init(ts);
+        return tmf;
+    }
+
 }


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@cassandra.apache.org
For additional commands, e-mail: commits-help@cassandra.apache.org