From commits-return-5427-archive-asf-public=cust-asf.ponee.io@kudu.apache.org Thu Mar 15 18:23:43 2018 Return-Path: X-Original-To: archive-asf-public@cust-asf.ponee.io Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by mx-eu-01.ponee.io (Postfix) with SMTP id 7AACA180649 for ; Thu, 15 Mar 2018 18:23:42 +0100 (CET) Received: (qmail 48868 invoked by uid 500); 15 Mar 2018 17:23:41 -0000 Mailing-List: contact commits-help@kudu.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@kudu.apache.org Delivered-To: mailing list commits@kudu.apache.org Received: (qmail 48859 invoked by uid 99); 15 Mar 2018 17:23:41 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Thu, 15 Mar 2018 17:23:41 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 74972F4EA4; Thu, 15 Mar 2018 17:23:41 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: todd@apache.org To: commits@kudu.apache.org Date: Thu, 15 Mar 2018 17:23:41 -0000 Message-Id: <65f89d753f0d4d3e8216477b2ad75ba2@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [1/2] kudu git commit: java: key ConnectionCache by address, improve stringification Repository: kudu Updated Branches: refs/heads/master f2479e21d -> ead756844 java: key ConnectionCache by address, improve stringification This changes the ConnectionCache in the Java client to be keyed by InetSocketAddress instead of server UUID. Although we currently assume that an address and UUID have a 1:1 correspondence, the logical purpose of the connection cache is to cache a connection to a specific server endpoint, and if a server were to re-register at a new IP, we'd really need to reconnect. Along the way this also adds and improves toString() methods for a number of core structures in the Java client. These new stringifications helped me debug a recent issue in the client where it wasn't clear that we were connecting to the wrong host. Change-Id: I431b5b6b8af6b81e6ab494d7f84fa2260bb0f941 Reviewed-on: http://gerrit.cloudera.org:8080/9643 Tested-by: Kudu Jenkins Reviewed-by: Grant Henke Project: http://git-wip-us.apache.org/repos/asf/kudu/repo Commit: http://git-wip-us.apache.org/repos/asf/kudu/commit/8204f43f Tree: http://git-wip-us.apache.org/repos/asf/kudu/tree/8204f43f Diff: http://git-wip-us.apache.org/repos/asf/kudu/diff/8204f43f Branch: refs/heads/master Commit: 8204f43f0204c172c4a799f9ccda0592d3b02138 Parents: f2479e2 Author: Todd Lipcon Authored: Wed Mar 14 14:05:34 2018 -0700 Committer: Grant Henke Committed: Thu Mar 15 16:13:09 2018 +0000 ---------------------------------------------------------------------- .../apache/kudu/client/AsyncKuduScanner.java | 2 +- .../java/org/apache/kudu/client/Connection.java | 5 +- .../org/apache/kudu/client/ConnectionCache.java | 31 +++++----- .../org/apache/kudu/client/RemoteTablet.java | 23 +++++++- .../java/org/apache/kudu/client/ServerInfo.java | 13 ++++- .../apache/kudu/client/TableLocationsCache.java | 30 +++++----- .../apache/kudu/client/TestConnectionCache.java | 30 +++------- .../apache/kudu/client/TestRemoteTablet.java | 50 +++++++++------- .../org/apache/kudu/client/TestServerInfo.java | 9 +-- .../kudu/client/TestTableLocationsCache.java | 61 ++++++++++++++++++++ 10 files changed, 167 insertions(+), 87 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduScanner.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduScanner.java b/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduScanner.java index 0863148..24581af 100644 --- a/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduScanner.java +++ b/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduScanner.java @@ -473,7 +473,7 @@ public final class AsyncKuduScanner { sequenceId = 0; return nextRows(); } else { - LOG.warn("Can not open scanner", e); + LOG.debug("Can not open scanner", e); // Don't let the scanner think it's opened on this tablet. return Deferred.fromError(e); // Let the error propogate. } http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/Connection.java ---------------------------------------------------------------------- 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 96395e4..b7513e4 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 @@ -17,7 +17,6 @@ package org.apache.kudu.client; -import java.net.InetSocketAddress; import java.nio.channels.ClosedChannelException; import java.util.ArrayList; import java.util.HashMap; @@ -483,7 +482,7 @@ class Connection extends SimpleChannelUpstreamHandler { /** @return string representation of the peer information suitable for logging */ String getLogPrefix() { - return "[peer " + serverInfo.getUuid() + "]"; + return "[peer " + serverInfo + "]"; } /** @@ -704,7 +703,7 @@ class Connection extends SimpleChannelUpstreamHandler { Preconditions.checkState(lock.isHeldByCurrentThread()); Preconditions.checkState(state == State.NEW); state = State.CONNECTING; - channel.connect(new InetSocketAddress(serverInfo.getResolvedAddress(), serverInfo.getPort())); + channel.connect(serverInfo.getResolvedAddress()); } /** Enumeration to represent the internal state of the Connection object. */ http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/ConnectionCache.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/ConnectionCache.java b/java/kudu-client/src/main/java/org/apache/kudu/client/ConnectionCache.java index db37425..c586886 100644 --- a/java/kudu-client/src/main/java/org/apache/kudu/client/ConnectionCache.java +++ b/java/kudu-client/src/main/java/org/apache/kudu/client/ConnectionCache.java @@ -17,6 +17,7 @@ package org.apache.kudu.client; +import java.net.InetSocketAddress; import java.util.ArrayList; import java.util.Iterator; import java.util.List; @@ -40,7 +41,7 @@ import org.jboss.netty.util.HashedWheelTimer; *

* Disconnected instances of the {@link Connection} class are replaced in the cache with new ones * when {@link #getConnection(ServerInfo, Connection.CredentialsPolicy)} method is called with the - * same destination and matching credentials policy. Since the map is keyed by the UUID of the + * same destination and matching credentials policy. Since the map is keyed by the address of the * target server, the theoretical maximum number of elements in the cache is twice the number of * all servers in the cluster (i.e. both masters and tablet servers). However, in practice it's * 2 * number of masters + number of tablet servers since tablet servers do not require connections @@ -65,15 +66,13 @@ class ConnectionCache { private final ClientSocketChannelFactory channelFactory; /** - * Container mapping server UUID into the established connection from the client to the server. - * It may be up to two connections per server: one established with secondary credentials - * (e.g. authn token), another with primary ones (e.g. Kerberos credentials). - * - * TODO(todd) it would make more sense to key this by IP address rather than by UUID in - * case a server actually changes address and re-registers to the cluster. + * Container mapping server IP/port into the established connection from the client to the + * server. It may be up to two connections per server: one established with secondary + * credentials (e.g. authn token), another with primary ones (e.g. Kerberos credentials). */ - @GuardedBy("uuid2connection") - private final HashMultimap uuid2connection = HashMultimap.create(); + @GuardedBy("connsByAddress") + private final HashMultimap connsByAddress = + HashMultimap.create(); /** Create a new empty ConnectionCache given the specified parameters. */ ConnectionCache(SecurityContext securityContext, @@ -98,7 +97,7 @@ class ConnectionCache { public Connection getConnection(final ServerInfo serverInfo, Connection.CredentialsPolicy credentialsPolicy) { Connection result = null; - synchronized (uuid2connection) { + synchronized (connsByAddress) { // Create and register a new connection object into the cache if one of the following is true: // // * There isn't a registered connection to the specified destination. @@ -115,7 +114,7 @@ class ConnectionCache { // special to the old connection to shut it down since it may be still in use. We rely // on the server to close inactive connections in accordance with their TTL settings. // - final Set connections = uuid2connection.get(serverInfo.getUuid()); + final Set connections = connsByAddress.get(serverInfo.getResolvedAddress()); Iterator it = connections.iterator(); while (it.hasNext()) { Connection c = it.next(); @@ -147,9 +146,9 @@ class ConnectionCache { /** Asynchronously terminate every connection. This cancels all the pending and in-flight RPCs. */ Deferred> disconnectEverything() { - synchronized (uuid2connection) { - List> deferreds = new ArrayList<>(uuid2connection.size()); - for (Connection c : uuid2connection.values()) { + synchronized (connsByAddress) { + List> deferreds = new ArrayList<>(connsByAddress.size()); + for (Connection c : connsByAddress.values()) { deferreds.add(c.shutdown()); } return Deferred.group(deferreds); @@ -165,8 +164,8 @@ class ConnectionCache { */ @VisibleForTesting List getConnectionListCopy() { - synchronized (uuid2connection) { - return ImmutableList.copyOf(uuid2connection.values()); + synchronized (connsByAddress) { + return ImmutableList.copyOf(connsByAddress.values()); } } } http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/RemoteTablet.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/RemoteTablet.java b/java/kudu-client/src/main/java/org/apache/kudu/client/RemoteTablet.java index a6025f7..bb2ca27 100644 --- a/java/kudu-client/src/main/java/org/apache/kudu/client/RemoteTablet.java +++ b/java/kudu-client/src/main/java/org/apache/kudu/client/RemoteTablet.java @@ -17,6 +17,8 @@ package org.apache.kudu.client; +import java.util.ArrayList; +import java.util.Collections; import java.util.HashMap; import java.util.List; import java.util.Map; @@ -24,6 +26,7 @@ import java.util.concurrent.atomic.AtomicReference; import javax.annotation.Nullable; import javax.annotation.concurrent.GuardedBy; +import com.google.common.base.Joiner; import com.google.common.base.Objects; import com.google.common.collect.ComparisonChain; import com.google.common.collect.ImmutableList; @@ -92,7 +95,22 @@ class RemoteTablet implements Comparable { @Override public String toString() { - return getTabletId(); + StringBuilder sb = new StringBuilder(); + sb.append(tabletId).append("@["); + List tsStrings; + synchronized (tabletServers) { + tsStrings = new ArrayList<>(tabletServers.size()); + for (ServerInfo e : tabletServers.values()) { + String flag = e.getUuid().equals(leaderUuid) ? "[L]" : ""; + tsStrings.add(e.toString() + flag); + } + } + // Sort so that we have a consistent iteration order regardless of + // HashSet ordering. + Collections.sort(tsStrings); + sb.append(Joiner.on(',').join(tsStrings)); + sb.append(']'); + return sb.toString(); } /** @@ -168,6 +186,9 @@ class RemoteTablet implements Comparable { return e; } } + // TODO(KUDU-2348) this doesn't return a random server, but rather returns + // whichever one's hashcode places it last. That might be the same + // "random" choice across all clients, which is not so good. return last; } } http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/ServerInfo.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/ServerInfo.java b/java/kudu-client/src/main/java/org/apache/kudu/client/ServerInfo.java index 4858a16..bed3e21 100644 --- a/java/kudu-client/src/main/java/org/apache/kudu/client/ServerInfo.java +++ b/java/kudu-client/src/main/java/org/apache/kudu/client/ServerInfo.java @@ -18,6 +18,7 @@ package org.apache.kudu.client; import java.net.InetAddress; +import java.net.InetSocketAddress; import java.net.UnknownHostException; import java.util.concurrent.ConcurrentHashMap; @@ -34,7 +35,7 @@ import org.apache.kudu.util.NetUtil; public class ServerInfo { private final String uuid; private final HostAndPort hostPort; - private final InetAddress resolvedAddr; + private final InetSocketAddress resolvedAddr; private final boolean local; private static final ConcurrentHashMap isLocalAddressCache = new ConcurrentHashMap<>(); @@ -48,9 +49,10 @@ public class ServerInfo { */ public ServerInfo(String uuid, HostAndPort hostPort, InetAddress resolvedAddr) { Preconditions.checkNotNull(uuid); + Preconditions.checkArgument(hostPort.getPort() > 0); this.uuid = uuid; this.hostPort = hostPort; - this.resolvedAddr = resolvedAddr; + this.resolvedAddr = new InetSocketAddress(resolvedAddr, hostPort.getPort()); Boolean isLocal = isLocalAddressCache.get(resolvedAddr); if (isLocal == null) { isLocal = NetUtil.isLocalAddress(resolvedAddr); @@ -98,7 +100,12 @@ public class ServerInfo { /** * @return the cached resolved address for this server */ - public InetAddress getResolvedAddress() { + public InetSocketAddress getResolvedAddress() { return resolvedAddr; } + + @Override + public String toString() { + return uuid + "(" + hostPort + ")"; + } } http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/main/java/org/apache/kudu/client/TableLocationsCache.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/TableLocationsCache.java b/java/kudu-client/src/main/java/org/apache/kudu/client/TableLocationsCache.java index eac658c..3e299ef 100644 --- a/java/kudu-client/src/main/java/org/apache/kudu/client/TableLocationsCache.java +++ b/java/kudu-client/src/main/java/org/apache/kudu/client/TableLocationsCache.java @@ -29,8 +29,10 @@ import java.util.concurrent.locks.ReentrantReadWriteLock; import javax.annotation.concurrent.GuardedBy; import javax.annotation.concurrent.ThreadSafe; +import com.google.common.annotations.VisibleForTesting; import com.google.common.base.MoreObjects; import com.google.common.base.Preconditions; +import com.google.common.base.Ticker; import com.google.common.primitives.UnsignedBytes; import org.apache.yetus.audience.InterfaceAudience; import org.slf4j.Logger; @@ -51,6 +53,9 @@ class TableLocationsCache { @GuardedBy("rwl") private final NavigableMap entries = new TreeMap<>(COMPARATOR); + @VisibleForTesting + static Ticker ticker = Ticker.systemTicker(); + public Entry get(byte[] partitionKey) { if (partitionKey == null) { @@ -97,7 +102,7 @@ class TableLocationsCache { byte[] requestPartitionKey, int requestedBatchSize, long ttl) { - long deadline = System.nanoTime() + ttl * TimeUnit.MILLISECONDS.toNanos(1); + long deadline = ticker.read() + ttl * TimeUnit.MILLISECONDS.toNanos(1); if (requestPartitionKey == null) { // Master lookup. Preconditions.checkArgument(tablets.size() == 1); @@ -290,7 +295,7 @@ class TableLocationsCache { } private long ttl() { - return TimeUnit.NANOSECONDS.toMillis(deadline - System.nanoTime()); + return TimeUnit.NANOSECONDS.toMillis(deadline - ticker.read()); } public boolean isStale() { @@ -299,20 +304,13 @@ class TableLocationsCache { @Override public String toString() { - if (isNonCoveredRange()) { - return MoreObjects.toStringHelper("NonCoveredRange") - .add("lowerBoundPartitionKey", Bytes.hex(lowerBoundPartitionKey)) - .add("upperBoundPartitionKey", Bytes.hex(upperBoundPartitionKey)) - .add("ttl", ttl()) - .toString(); - } else { - return MoreObjects.toStringHelper("Tablet") - .add("lowerBoundPartitionKey", Bytes.hex(getLowerBoundPartitionKey())) - .add("upperBoundPartitionKey", Bytes.hex(getUpperBoundPartitionKey())) - .add("tablet-id", tablet.getTabletId()) - .add("ttl", ttl()) - .toString(); - } + return MoreObjects.toStringHelper(isNonCoveredRange() ? "NonCoveredRange" : "Tablet") + .omitNullValues() + .add("lowerBoundPartitionKey", Bytes.hex(getLowerBoundPartitionKey())) + .add("upperBoundPartitionKey", Bytes.hex(getUpperBoundPartitionKey())) + .add("ttl", ttl()) + .add("tablet", tablet) + .toString(); } } } http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/test/java/org/apache/kudu/client/TestConnectionCache.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestConnectionCache.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestConnectionCache.java index 6759d59..84dacbf 100644 --- a/java/kudu-client/src/test/java/org/apache/kudu/client/TestConnectionCache.java +++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestConnectionCache.java @@ -22,9 +22,6 @@ import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertNotSame; import static org.junit.Assert.assertTrue; -import java.util.List; - -import com.google.common.collect.Lists; import com.google.common.net.HostAndPort; import com.stumbleupon.async.Deferred; import org.junit.Test; @@ -41,34 +38,23 @@ public class TestConnectionCache { final AsyncKuduClient client = new AsyncKuduClient.AsyncKuduClientBuilder(cluster.getMasterAddresses()).build(); - final List addresses = cluster.getMasterHostPorts(); - // Below we ping the masters directly using RpcProxy, so if they aren't ready to process // RPCs we'll get an error. Here by listing the tables we make sure this won't happen since // it won't return until a master leader is found. client.getTablesList().join(); - final List serverInfos = Lists.newArrayList(); - int i = 0; - for (HostAndPort hp : addresses) { - serverInfos.add(new ServerInfo("" + i, hp, NetUtil.getInetAddress(hp.getHost()))); - ++i; - } - - // Ping the process so we go through the whole connection process. - for (ServerInfo si : serverInfos) { - final RpcProxy h = client.newRpcProxy(si); - assertNotNull(h.getConnection()); - pingConnection(h); - } + HostAndPort masterHostPort = cluster.getMasterHostPorts().get(0); + ServerInfo firstMaster = new ServerInfo("fake-uuid", masterHostPort, + NetUtil.getInetAddress(masterHostPort.getHost())); - // 3 masters and 3 connections from the newRpcProxy() in the loop above. + // 3 masters in the cluster. Connections should have been cached since we forced + // a cluster connection above. // No tservers have been connected to by the client since we haven't accessed // any data. - assertEquals(3 + 3, client.getConnectionListCopy().size()); + assertEquals(3, client.getConnectionListCopy().size()); assertFalse(allConnectionsTerminated(client)); - final RpcProxy proxy = client.newRpcProxy(serverInfos.get(0)); + final RpcProxy proxy = client.newRpcProxy(firstMaster); // Disconnect from the server. proxy.getConnection().disconnect().awaitUninterruptibly(); @@ -80,7 +66,7 @@ public class TestConnectionCache { assertFalse(allConnectionsTerminated(client)); // For a new RpcProxy instance, a new connection to the same destination is established. - final RpcProxy newHelper = client.newRpcProxy(serverInfos.get(0)); + final RpcProxy newHelper = client.newRpcProxy(firstMaster); final Connection newConnection = newHelper.getConnection(); assertNotNull(newConnection); assertNotSame(proxy.getConnection(), newConnection); http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/test/java/org/apache/kudu/client/TestRemoteTablet.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestRemoteTablet.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestRemoteTablet.java index 5516648..9ba6d00 100644 --- a/java/kudu-client/src/test/java/org/apache/kudu/client/TestRemoteTablet.java +++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestRemoteTablet.java @@ -35,33 +35,34 @@ import org.apache.kudu.consensus.Metadata; import org.apache.kudu.master.Master; public class TestRemoteTablet { + private static final String[] kUuids = { "uuid-0", "uuid-1", "uuid-2" }; @Test public void testLeaderLastRemovedLast() { RemoteTablet tablet = getTablet(2); // Demote the wrong leader, no-op. - assertEquals("2", tablet.getLeaderServerInfo().getUuid()); - tablet.demoteLeader("1"); - assertEquals("2", tablet.getLeaderServerInfo().getUuid()); + assertEquals(kUuids[2], tablet.getLeaderServerInfo().getUuid()); + tablet.demoteLeader(kUuids[1]); + assertEquals(kUuids[2], tablet.getLeaderServerInfo().getUuid()); // Tablet at server 1 was deleted. - assertTrue(tablet.removeTabletClient("1")); - assertEquals("2", tablet.getLeaderServerInfo().getUuid()); + assertTrue(tablet.removeTabletClient(kUuids[1])); + assertEquals(kUuids[2], tablet.getLeaderServerInfo().getUuid()); // Simulate another thread trying to remove 1. - assertFalse(tablet.removeTabletClient("1")); + assertFalse(tablet.removeTabletClient(kUuids[1])); // Tablet at server 0 was deleted. - assertTrue(tablet.removeTabletClient("0")); - assertEquals("2", tablet.getLeaderServerInfo().getUuid()); + assertTrue(tablet.removeTabletClient(kUuids[0])); + assertEquals(kUuids[2], tablet.getLeaderServerInfo().getUuid()); // Leader was demoted. - tablet.demoteLeader("2"); + tablet.demoteLeader(kUuids[2]); assertNull(tablet.getLeaderServerInfo()); // Simulate another thread doing the same. - tablet.demoteLeader("2"); + tablet.demoteLeader(kUuids[2]); assertNull(tablet.getLeaderServerInfo()); } @@ -70,11 +71,11 @@ public class TestRemoteTablet { RemoteTablet tablet = getTablet(2); // Test we can remove it. - assertTrue(tablet.removeTabletClient("2")); + assertTrue(tablet.removeTabletClient("uuid-2")); assertNull(tablet.getLeaderServerInfo()); // Test demoting it doesn't break anything. - tablet.demoteLeader("2"); + tablet.demoteLeader("uuid-2"); assertNull(tablet.getLeaderServerInfo()); } @@ -83,22 +84,22 @@ public class TestRemoteTablet { RemoteTablet tablet = getTablet(0); // Test we can remove it. - assertTrue(tablet.removeTabletClient("0")); + assertTrue(tablet.removeTabletClient("uuid-0")); assertNull(tablet.getLeaderServerInfo()); // Test demoting it doesn't break anything. - tablet.demoteLeader("0"); + tablet.demoteLeader("uuid-0"); assertNull(tablet.getLeaderServerInfo()); // Test removing a server with no leader doesn't break. - assertTrue(tablet.removeTabletClient("2")); + assertTrue(tablet.removeTabletClient("uuid-2")); } @Test public void testLocalReplica() { RemoteTablet tablet = getTablet(0, 0); - assertEquals("0", tablet.getClosestServerInfo().getUuid()); + assertEquals(kUuids[0], tablet.getClosestServerInfo().getUuid()); } @Test @@ -113,17 +114,24 @@ public class TestRemoteTablet { public void testReplicaSelection() { RemoteTablet tablet = getTablet(0, 1); - assertEquals("0", + assertEquals(kUuids[0], tablet.getReplicaSelectedServerInfo(ReplicaSelection.LEADER_ONLY).getUuid()); - assertEquals("1", + assertEquals(kUuids[1], tablet.getReplicaSelectedServerInfo(ReplicaSelection.CLOSEST_REPLICA).getUuid()); } + @Test + public void testToString() { + RemoteTablet tablet = getTablet(0, 1); + assertEquals("fake tablet@[uuid-0(host:1000)[L],uuid-1(host:1001),uuid-2(host:1002)]", + tablet.toString()); + } + private RemoteTablet getTablet(int leaderIndex) { return getTablet(leaderIndex, -1); } - private RemoteTablet getTablet(int leaderIndex, int localReplicaIndex) { + static RemoteTablet getTablet(int leaderIndex, int localReplicaIndex) { Master.TabletLocationsPB.Builder tabletPb = Master.TabletLocationsPB.newBuilder(); tabletPb.setPartition(TestUtils.getFakePartitionPB()); @@ -141,9 +149,9 @@ public class TestRemoteTablet { throw new RuntimeException(e); } - String uuid = i + ""; + String uuid = kUuids[i]; servers.add(new ServerInfo(uuid, - HostAndPort.fromParts("host", i), + HostAndPort.fromParts("host", 1000 + i), addr)); tabletPb.addReplicas(TestUtils.getFakeTabletReplicaPB( uuid, "host", i, http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/test/java/org/apache/kudu/client/TestServerInfo.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestServerInfo.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestServerInfo.java index a26e4f2..38df64d 100644 --- a/java/kudu-client/src/test/java/org/apache/kudu/client/TestServerInfo.java +++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestServerInfo.java @@ -28,7 +28,7 @@ public class TestServerInfo { @Test(timeout = 500) public void testConstructorNotSlow() throws Exception { String uuid = "nevermind"; - HostAndPort hap = HostAndPort.fromString("nevermind"); + HostAndPort hap = HostAndPort.fromString("nevermind:12345"); // ip to force NetworkInterface.getByInetAddress call InetAddress ia = InetAddress.getByName("8.8.8.8"); for (int i = 0; i < 100; ++i) { @@ -45,7 +45,7 @@ public class TestServerInfo { ServerInfo serverInfo = new ServerInfo( "nevermind", - HostAndPort.fromHost("master2.example.com"), + HostAndPort.fromParts("master2.example.com", 12345), InetAddress.getByName("10.1.2.3")); Assert.assertEquals("master2.example.com", serverInfo.getAndCanonicalizeHostname()); @@ -60,11 +60,12 @@ public class TestServerInfo { installFakeDNS("master1.example.com", "server123.example.com", "10.1.2.3"); ServerInfo serverInfo = new ServerInfo( - "nevermind", - HostAndPort.fromHost("master1.example.com"), + "abcdef", // uuid + HostAndPort.fromParts("master1.example.com", 12345), InetAddress.getByName("10.1.2.3")); Assert.assertEquals("server123.example.com", serverInfo.getAndCanonicalizeHostname()); + Assert.assertEquals("abcdef(master1.example.com:12345)", serverInfo.toString()); } /** http://git-wip-us.apache.org/repos/asf/kudu/blob/8204f43f/java/kudu-client/src/test/java/org/apache/kudu/client/TestTableLocationsCache.java ---------------------------------------------------------------------- diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestTableLocationsCache.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestTableLocationsCache.java new file mode 100644 index 0000000..d61fb8e --- /dev/null +++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestTableLocationsCache.java @@ -0,0 +1,61 @@ +// 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.kudu.client; + +import static org.junit.Assert.*; + +import java.util.List; + +import org.junit.After; +import org.junit.Before; +import org.junit.Test; +import org.mockito.Mockito; + +import com.google.common.base.Ticker; +import com.google.common.collect.ImmutableList; + +public class TestTableLocationsCache { + private TableLocationsCache cache = new TableLocationsCache(); + + /** + * Prevent time from advancing during the test by mocking the time. + */ + @Before + public void mockTime() { + TableLocationsCache.ticker = Mockito.mock(Ticker.class); + } + @After + public void unmockTime() { + TableLocationsCache.ticker = Ticker.systemTicker(); + } + + @Test + public void testToString() { + RemoteTablet tablet = TestRemoteTablet.getTablet(0, 1); + List tablets = ImmutableList.of(tablet); + cache.cacheTabletLocations(tablets, + tablet.getPartition().getPartitionKeyStart(), + 1, // requested batch size, + 100); // ttl + // Mock as if the time increased by 10ms (the ticker is in nanoseconds). + // This will result in a remaining TTL of 90. + Mockito.when(TableLocationsCache.ticker.read()).thenReturn(10 * 1000000L); + assertEquals("[Tablet{lowerBoundPartitionKey=0x, upperBoundPartitionKey=0x, " + + "ttl=90, tablet=" + tablet.toString() + "}]", + cache.toString()); + } +}