You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kudu.apache.org by al...@apache.org on 2019/11/16 01:41:23 UTC

[kudu] branch branch-1.10.x updated (b7e2fae -> d7e2e99)

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

alexey pushed a change to branch branch-1.10.x
in repository https://gitbox.apache.org/repos/asf/kudu.git.


    from b7e2fae  thirdparty: remove memkind from LICENSE.txt
     new 2f027c3  [mini_cluster] SetDaemonFlag for test minicluster
     new d7e2e99  [java] fixed bug in the connection negotiation code

The 2 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 .../java/org/apache/kudu/client/Connection.java    |  6 ++
 .../java/org/apache/kudu/client/Negotiator.java    |  3 +
 .../java/org/apache/kudu/client/TestSecurity.java  | 58 +++++++++++++++++++-
 .../apache/kudu/test/cluster/MiniKuduCluster.java  | 39 +++++++++++++
 src/kudu/rpc/server_negotiation.cc                 | 20 ++++++-
 src/kudu/tools/kudu-tool-test.cc                   | 64 ++++++++++++++++++++++
 src/kudu/tools/tool.proto                          | 10 ++++
 src/kudu/tools/tool_action_test.cc                 | 17 ++++++
 8 files changed, 214 insertions(+), 3 deletions(-)


[kudu] 01/02: [mini_cluster] SetDaemonFlag for test minicluster

Posted by al...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

alexey pushed a commit to branch branch-1.10.x
in repository https://gitbox.apache.org/repos/asf/kudu.git

commit 2f027c31ceb174c99ba0eea53df81b8ac6baec12
Author: Alexey Serbin <al...@apache.org>
AuthorDate: Thu Nov 14 19:45:30 2019 -0800

    [mini_cluster] SetDaemonFlag for test minicluster
    
    Added control knobs to call SetFlag() RPC on masters and tablet servers
    via mini_cluster test interface.  Also, added a test to cover the
    new functionality.
    
    Change-Id: Ie05aa87bba1b709cbbab953b6b220cae0fc958bb
    Reviewed-on: http://gerrit.cloudera.org:8080/14712
    Tested-by: Kudu Jenkins
    Reviewed-by: Adar Dembo <ad...@cloudera.com>
    (cherry picked from commit 9b1ff304ca5f9f2e1ea08646c7307fcbe0486015)
    Reviewed-on: http://gerrit.cloudera.org:8080/14723
    Reviewed-by: Alexey Serbin <as...@cloudera.com>
---
 .../apache/kudu/test/cluster/MiniKuduCluster.java  | 39 +++++++++++++
 src/kudu/tools/kudu-tool-test.cc                   | 64 ++++++++++++++++++++++
 src/kudu/tools/tool.proto                          | 10 ++++
 src/kudu/tools/tool_action_test.cc                 | 17 ++++++
 4 files changed, 130 insertions(+)

diff --git a/java/kudu-test-utils/src/main/java/org/apache/kudu/test/cluster/MiniKuduCluster.java b/java/kudu-test-utils/src/main/java/org/apache/kudu/test/cluster/MiniKuduCluster.java
index 2d40df6..ff5a15a 100644
--- a/java/kudu-test-utils/src/main/java/org/apache/kudu/test/cluster/MiniKuduCluster.java
+++ b/java/kudu-test-utils/src/main/java/org/apache/kudu/test/cluster/MiniKuduCluster.java
@@ -52,6 +52,7 @@ import org.apache.kudu.tools.Tool.GetMastersRequestPB;
 import org.apache.kudu.tools.Tool.GetTServersRequestPB;
 import org.apache.kudu.tools.Tool.KdestroyRequestPB;
 import org.apache.kudu.tools.Tool.KinitRequestPB;
+import org.apache.kudu.tools.Tool.SetDaemonFlagRequestPB;
 import org.apache.kudu.tools.Tool.StartClusterRequestPB;
 import org.apache.kudu.tools.Tool.StartDaemonRequestPB;
 import org.apache.kudu.tools.Tool.StopDaemonRequestPB;
@@ -421,6 +422,44 @@ public class MiniKuduCluster implements AutoCloseable {
   }
 
   /**
+   * Set flag for the specified master.
+   *
+   * @param hp unique host and port identifying the target master
+   * @throws IOException if something went wrong in transit
+   */
+  public void setMasterFlag(HostAndPort hp, String flag, String value)
+      throws IOException {
+    DaemonInfo d = getMasterServer(hp);
+    LOG.info("Setting flag for master at {}", hp);
+    sendRequestToCluster(ControlShellRequestPB.newBuilder()
+        .setSetDaemonFlag(SetDaemonFlagRequestPB.newBuilder()
+            .setId(d.id)
+            .setFlag(flag)
+            .setValue(value)
+            .build())
+        .build());
+  }
+
+  /**
+   * Set flag for the specified tablet server.
+   *
+   * @param hp unique host and port identifying the target tablet server
+   * @throws IOException if something went wrong in transit
+   */
+  public void setTServerFlag(HostAndPort hp, String flag, String value)
+      throws IOException {
+    DaemonInfo d = getTabletServer(hp);
+    LOG.info("Setting flag for tserver at {}", hp);
+    sendRequestToCluster(ControlShellRequestPB.newBuilder()
+        .setSetDaemonFlag(SetDaemonFlagRequestPB.newBuilder()
+            .setId(d.id)
+            .setFlag(flag)
+            .setValue(value)
+            .build())
+        .build());
+  }
+
+  /**
    * Removes all credentials for all principals from the Kerberos credential cache.
    */
   public void kdestroy() throws IOException {
diff --git a/src/kudu/tools/kudu-tool-test.cc b/src/kudu/tools/kudu-tool-test.cc
index be454f6..42527cd 100644
--- a/src/kudu/tools/kudu-tool-test.cc
+++ b/src/kudu/tools/kudu-tool-test.cc
@@ -4335,6 +4335,70 @@ TEST_P(ControlShellToolTest, TestControlShell) {
     ASSERT_OK(SendReceive(req, &resp));
   }
 
+  // Set flag.
+  {
+    ControlShellRequestPB req;
+    ControlShellResponsePB resp;
+    auto* r = req.mutable_set_daemon_flag();
+    *r->mutable_id() = tservers[0].id();
+    r->set_flag("rpc_negotiation_timeout_ms");
+    r->set_value("5000");
+    ASSERT_OK(SendReceive(req, &resp));
+  }
+
+  // Try to set a non-existent flag: this should fail.
+  {
+    ControlShellRequestPB req;
+    ControlShellResponsePB resp;
+    auto* r = req.mutable_set_daemon_flag();
+    *r->mutable_id() = masters[0].id();
+    r->set_flag("__foo_bar_flag__");
+    r->set_value("__value__");
+    ASSERT_OK(proto_->SendMessage(req));
+    ASSERT_OK(proto_->ReceiveMessage(&resp));
+    ASSERT_TRUE(resp.has_error());
+    auto s = StatusFromPB(resp.error());
+    ASSERT_TRUE(s.IsRemoteError()) << s.ToString();
+    ASSERT_STR_CONTAINS(s.ToString(),
+                        "failed to set flag: result: NO_SUCH_FLAG");
+  }
+
+  // Try to set a flag on a non-existent daemon: this should fail.
+  {
+    ControlShellRequestPB req;
+    ControlShellResponsePB resp;
+    auto* r = req.mutable_set_daemon_flag();
+    r->mutable_id()->set_index(1000);
+    r->mutable_id()->set_type(MASTER);
+    r->set_flag("flag");
+    r->set_value("value");
+    ASSERT_OK(proto_->SendMessage(req));
+    ASSERT_OK(proto_->ReceiveMessage(&resp));
+    ASSERT_TRUE(resp.has_error());
+    auto s = StatusFromPB(resp.error());
+    ASSERT_TRUE(s.IsNotFound()) << s.ToString();
+    ASSERT_STR_CONTAINS(s.ToString(), "no master with index 1000");
+  }
+
+  // Try to set a flag on a KDC: this should fail since mini-KDC doesn't support
+  // SetFlag() call.
+  {
+    ControlShellRequestPB req;
+    ControlShellResponsePB resp;
+    auto* r = req.mutable_set_daemon_flag();
+    r->mutable_id()->set_index(0);
+    r->mutable_id()->set_type(KDC);
+    r->set_flag("flag");
+    r->set_value("value");
+    ASSERT_OK(proto_->SendMessage(req));
+    ASSERT_OK(proto_->ReceiveMessage(&resp));
+    ASSERT_TRUE(resp.has_error());
+    auto s = StatusFromPB(resp.error());
+    ASSERT_TRUE(s.IsInvalidArgument()) << s.ToString();
+    ASSERT_STR_CONTAINS(s.ToString(),
+                        "mini-KDC doesn't support SetFlag()");
+  }
+
   if (enable_kerberos()) {
     // Restart the KDC.
     {
diff --git a/src/kudu/tools/tool.proto b/src/kudu/tools/tool.proto
index 2a9ff89..629b659 100644
--- a/src/kudu/tools/tool.proto
+++ b/src/kudu/tools/tool.proto
@@ -159,6 +159,15 @@ message KinitRequestPB {
   optional string username = 1 [ default = "test-admin" ];
 };
 
+// Call SetFlag() on the specific daemon.
+message SetDaemonFlagRequestPB {
+  // The identifier for the daemon to sent the request to.
+  optional DaemonIdentifierPB id = 1;
+  // The name of the flag to set.
+  optional string flag = 2;
+  // Value to set.
+  optional string value = 3;
+}
 
 // Sent by the control shell in response to a control shell command request.
 message ControlShellResponsePB {
@@ -194,6 +203,7 @@ message ControlShellRequestPB {
     GetKDCEnvVarsRequestPB get_kdc_env_vars = 9;
     KdestroyRequestPB kdestroy = 10;
     KinitRequestPB kinit = 11;
+    SetDaemonFlagRequestPB set_daemon_flag = 12;
   }
 }
 
diff --git a/src/kudu/tools/tool_action_test.cc b/src/kudu/tools/tool_action_test.cc
index af65dad..da0de8d 100644
--- a/src/kudu/tools/tool_action_test.cc
+++ b/src/kudu/tools/tool_action_test.cc
@@ -286,6 +286,23 @@ Status ProcessRequest(const ControlShellRequestPB& req,
       RETURN_NOT_OK((*cluster)->kdc()->Kinit(req.kinit().username()));
       break;
     }
+    case ControlShellRequestPB::kSetDaemonFlag:
+    {
+      const auto& r = req.set_daemon_flag();
+      if (!r.has_id()) {
+        RETURN_NOT_OK(Status::InvalidArgument("missing process id"));
+      }
+      const auto& id = r.id();
+      if (id.type() == DaemonType::KDC) {
+        return Status::InvalidArgument("mini-KDC doesn't support SetFlag()");
+      }
+      ExternalDaemon* daemon;
+      MiniKdc* kdc;
+      RETURN_NOT_OK(FindDaemon(*cluster, id, &daemon, &kdc));
+      DCHECK(daemon);
+      RETURN_NOT_OK((*cluster)->SetFlag(daemon, r.flag(), r.value()));
+      break;
+    }
     default:
       RETURN_NOT_OK(Status::InvalidArgument("unknown cluster control request"));
   }


[kudu] 02/02: [java] fixed bug in the connection negotiation code

Posted by al...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

alexey pushed a commit to branch branch-1.10.x
in repository https://gitbox.apache.org/repos/asf/kudu.git

commit d7e2e998cf963dc98c568f264cc09c7c320cae0a
Author: Alexey Serbin <al...@apache.org>
AuthorDate: Thu Nov 14 19:26:01 2019 -0800

    [java] fixed bug in the connection negotiation code
    
    This patch fixes a typo in the connection negotiation code in the Java
    client.  Prior to this fix, channel binding information was not verified
    during connection negotiation because the peer certificate was not set.
    
    In addition, I modified the error handing code in Negotiator.java to
    abort connection negotiation upon receiving SSLPeerUnverifiedException
    due to the absence of the channel binding information or the presence
    of the invalid one.
    
    I also added a test to verify that Kudu Java client doesn't connect
    to a Kudu server which doesn't provide valid channel binding information
    during the connection negotiation phase.
    
    Kudos to Andy Singer for pointing to the bug.
    
    Change-Id: I7bfd428128e224f03901a6cd7b33283495a28d54
    Reviewed-on: http://gerrit.cloudera.org:8080/14713
    Tested-by: Kudu Jenkins
    Reviewed-by: Adar Dembo <ad...@cloudera.com>
    Reviewed-by: Todd Lipcon <to...@apache.org>
    (cherry picked from commit a0e896475c139d308e3b6e32110e97168b9562c6)
    Reviewed-on: http://gerrit.cloudera.org:8080/14724
    Reviewed-by: Alexey Serbin <as...@cloudera.com>
---
 .../java/org/apache/kudu/client/Connection.java    |  6 +++
 .../java/org/apache/kudu/client/Negotiator.java    |  3 ++
 .../java/org/apache/kudu/client/TestSecurity.java  | 58 +++++++++++++++++++++-
 src/kudu/rpc/server_negotiation.cc                 | 20 +++++++-
 4 files changed, 84 insertions(+), 3 deletions(-)

diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/Connection.java b/java/kudu-client/src/main/java/org/apache/kudu/client/Connection.java
index 5da0a8c..d02d27f 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/Connection.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/Connection.java
@@ -30,6 +30,7 @@ import java.util.concurrent.locks.ReentrantLock;
 import javax.annotation.Nullable;
 import javax.annotation.concurrent.GuardedBy;
 import javax.net.ssl.SSLException;
+import javax.net.ssl.SSLPeerUnverifiedException;
 
 import com.google.common.base.Preconditions;
 import com.google.common.collect.Lists;
@@ -451,6 +452,11 @@ class Connection extends SimpleChannelUpstreamHandler {
       // SSLException if we've already attempted to close, otherwise log the error.
       error = new RecoverableException(Status.NetworkError(
           String.format("%s disconnected from peer", getLogPrefix())));
+    } else if (e instanceof SSLPeerUnverifiedException) {
+      String m = String.format("unable to verify identity of peer %s: %s",
+          serverInfo, e.getMessage());
+      error = new NonRecoverableException(Status.NetworkError(m), e);
+      LOG.error(m, e);
     } else {
       // If the connection was explicitly disconnected via a call to disconnect(), we should
       // have either gotten a ClosedChannelException or an SSLException.
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/Negotiator.java b/java/kudu-client/src/main/java/org/apache/kudu/client/Negotiator.java
index 9168031..17a7442 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/Negotiator.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/Negotiator.java
@@ -601,6 +601,9 @@ public class Negotiator extends SimpleChannelUpstreamHandler {
       throw new SSLPeerUnverifiedException("no peer cert found");
     }
 
+    // The first element of the array is the peer's own certificate.
+    peerCert = certs[0];
+
     // Don't wrap the TLS socket if we are using TLS for authentication only.
     boolean isAuthOnly = serverFeatures.contains(RpcFeatureFlag.TLS_AUTHENTICATION_ONLY) &&
         isLoopbackConnection(chan);
diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestSecurity.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestSecurity.java
index 3c4de25..37489a8 100644
--- a/java/kudu-client/src/test/java/org/apache/kudu/client/TestSecurity.java
+++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestSecurity.java
@@ -30,6 +30,7 @@ import javax.security.auth.Subject;
 
 import com.google.common.base.Stopwatch;
 import com.google.common.collect.ImmutableSet;
+import com.google.common.collect.ImmutableList;
 import com.stumbleupon.async.Deferred;
 import org.hamcrest.CoreMatchers;
 import org.junit.After;
@@ -61,8 +62,19 @@ public class TestSecurity {
   private enum Option {
     LONG_LEADER_ELECTION,
     SHORT_TOKENS_AND_TICKETS,
-    START_TSERVERS
-  };
+    START_TSERVERS,
+  }
+
+  static private class KeyValueMessage {
+    final String key;
+    final String val;
+    final String msg;
+    KeyValueMessage(String k, String v, String m) {
+      key = k;
+      val = v;
+      msg = m;
+    }
+  }
 
   private void startCluster(Set<Option> opts) throws IOException {
     MiniKuduClusterBuilder mcb = new MiniKuduClusterBuilder();
@@ -435,4 +447,46 @@ public class TestSecurity {
     Assert.assertThat(cla.getAppendedText(), CoreMatchers.containsString(
         "Using caller-provided subject with Kerberos principal test-admin@KRBTEST.COM."));
   }
+
+  /**
+   * Test that if a Kudu server (in this case master) doesn't provide valid
+   * connection binding information, Java client fails to connect to the server.
+   */
+  @Test(timeout=60000)
+  public void testNegotiationChannelBindings() throws Exception {
+    startCluster(ImmutableSet.of(Option.START_TSERVERS));
+    // Test precondition: all is well with masters -- the client is able
+    // to connect to the cluster and create a table.
+    client.createTable("TestSecurity-channel-bindings-0",
+        getBasicSchema(), getBasicCreateTableOptions());
+
+    List<KeyValueMessage> variants = ImmutableList.of(
+        new KeyValueMessage("rpc_inject_invalid_channel_bindings_ratio", "1.0",
+            "invalid channel bindings provided by remote peer"),
+        new KeyValueMessage("rpc_send_channel_bindings", "false",
+            "no channel bindings provided by remote peer"));
+
+    // Make all masters sending invalid channel binding info during connection
+    // negotiation.
+    for (KeyValueMessage kvm : variants) {
+      for (HostAndPort hp : miniCluster.getMasterServers()) {
+        miniCluster.setMasterFlag(hp, kvm.key, kvm.val);
+      }
+
+      // Now, a client should not be able to connect to any master: negotiation
+      // fails because client cannot authenticate the servers since it fails
+      // to verify the connection binding.
+      try {
+        KuduClient c = new KuduClient.KuduClientBuilder(
+            miniCluster.getMasterAddressesAsString()).build();
+        c.createTable("TestSecurity-channel-bindings-1",
+            getBasicSchema(), getBasicCreateTableOptions());
+        Assert.fail("client should not be able to connect to any master");
+      } catch (NonRecoverableException e) {
+        Assert.assertThat(e.getMessage(), CoreMatchers.containsString(
+            "unable to verify identity of peer"));
+        Assert.assertThat(e.getMessage(), CoreMatchers.containsString(kvm.msg));
+      }
+    }
+  }
 }
diff --git a/src/kudu/rpc/server_negotiation.cc b/src/kudu/rpc/server_negotiation.cc
index 04e08b1..15c47d7 100644
--- a/src/kudu/rpc/server_negotiation.cc
+++ b/src/kudu/rpc/server_negotiation.cc
@@ -80,6 +80,18 @@ DEFINE_double(rpc_inject_invalid_authn_token_ratio, 0,
 TAG_FLAG(rpc_inject_invalid_authn_token_ratio, runtime);
 TAG_FLAG(rpc_inject_invalid_authn_token_ratio, unsafe);
 
+DEFINE_double(rpc_inject_invalid_channel_bindings_ratio, 0,
+            "The ratio of injection of invalid channel bindings during "
+            "connection negotiation. This is a test-only flag.");
+TAG_FLAG(rpc_inject_invalid_channel_bindings_ratio, runtime);
+TAG_FLAG(rpc_inject_invalid_channel_bindings_ratio, unsafe);
+
+DEFINE_bool(rpc_send_channel_bindings, true,
+            "Whether to send channel bindings in NegotiatePB response as "
+            "prescribed by RFC 5929. This is a test-only flag.");
+TAG_FLAG(rpc_send_channel_bindings, runtime);
+TAG_FLAG(rpc_send_channel_bindings, unsafe);
+
 DECLARE_bool(rpc_encrypt_loopback_connections);
 
 DEFINE_string(trusted_subnets,
@@ -866,7 +878,7 @@ Status ServerNegotiation::SendSaslSuccess() {
     RETURN_NOT_OK(security::GenerateNonce(nonce_.get_ptr()));
     response.set_nonce(*nonce_);
 
-    if (tls_negotiated_) {
+    if (tls_negotiated_ && PREDICT_TRUE(FLAGS_rpc_send_channel_bindings)) {
       // Send the channel bindings to the client.
       security::Cert cert;
       RETURN_NOT_OK(tls_handshake_.GetLocalCert(&cert));
@@ -874,6 +886,12 @@ Status ServerNegotiation::SendSaslSuccess() {
       string plaintext_channel_bindings;
       RETURN_NOT_OK(cert.GetServerEndPointChannelBindings(&plaintext_channel_bindings));
 
+      if (kudu::fault_injection::MaybeTrue(
+          FLAGS_rpc_inject_invalid_channel_bindings_ratio)) {
+        DCHECK_GT(plaintext_channel_bindings.size(), 0);
+        plaintext_channel_bindings[0] += 1;
+      }
+
       Slice ciphertext;
       RETURN_NOT_OK(SaslEncode(sasl_conn_.get(),
                                plaintext_channel_bindings,