You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by sl...@apache.org on 2016/01/07 11:24:38 UTC
[2/3] cassandra git commit: Commit CASSANDRA-9258 to 3.3+
Commit CASSANDRA-9258 to 3.3+
For some reason I don't fully understand, the patch for CASSANRA-9258
didn't made it from 3.0 to the 3.3 and trunk branches so this commit
it.
Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo
Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/ffc0840d
Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/ffc0840d
Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/ffc0840d
Branch: refs/heads/trunk
Commit: ffc0840d03014331035f24be3c76aa812973089f
Parents: 65b1f2d
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Thu Jan 7 11:23:05 2016 +0100
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Thu Jan 7 11:23:05 2016 +0100
----------------------------------------------------------------------
.../apache/cassandra/locator/TokenMetadata.java | 61 ++++++++++----------
.../cassandra/service/StorageService.java | 2 +-
2 files changed, 32 insertions(+), 31 deletions(-)
----------------------------------------------------------------------
http://git-wip-us.apache.org/repos/asf/cassandra/blob/ffc0840d/src/java/org/apache/cassandra/locator/TokenMetadata.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/locator/TokenMetadata.java b/src/java/org/apache/cassandra/locator/TokenMetadata.java
index a3be9de..f1348c1 100644
--- a/src/java/org/apache/cassandra/locator/TokenMetadata.java
+++ b/src/java/org/apache/cassandra/locator/TokenMetadata.java
@@ -84,7 +84,7 @@ public class TokenMetadata
// (don't need to record Token here since it's still part of tokenToEndpointMap until it's done leaving)
private final Set<InetAddress> leavingEndpoints = new HashSet<>();
// this is a cache of the calculation from {tokenToEndpointMap, bootstrapTokens, leavingEndpoints}
- private final ConcurrentMap<String, Multimap<Range<Token>, InetAddress>> pendingRanges = new ConcurrentHashMap<>();
+ private final ConcurrentMap<String, PendingRangeMaps> pendingRanges = new ConcurrentHashMap<String, PendingRangeMaps>();
// nodes which are migrating to the new tokens in the ring
private final Set<Pair<Token, InetAddress>> movingEndpoints = new HashSet<>();
@@ -688,23 +688,30 @@ public class TokenMetadata
return sortedTokens;
}
- private Multimap<Range<Token>, InetAddress> getPendingRangesMM(String keyspaceName)
+ public Multimap<Range<Token>, InetAddress> getPendingRangesMM(String keyspaceName)
{
- Multimap<Range<Token>, InetAddress> map = pendingRanges.get(keyspaceName);
- if (map == null)
+ Multimap<Range<Token>, InetAddress> map = HashMultimap.create();
+ PendingRangeMaps pendingRangeMaps = this.pendingRanges.get(keyspaceName);
+
+ if (pendingRangeMaps != null)
{
- map = HashMultimap.create();
- Multimap<Range<Token>, InetAddress> priorMap = pendingRanges.putIfAbsent(keyspaceName, map);
- if (priorMap != null)
- map = priorMap;
+ for (Map.Entry<Range<Token>, List<InetAddress>> entry : pendingRangeMaps)
+ {
+ Range<Token> range = entry.getKey();
+ for (InetAddress address : entry.getValue())
+ {
+ map.put(range, address);
+ }
+ }
}
+
return map;
}
/** a mutable map may be returned but caller should not modify it */
- public Map<Range<Token>, Collection<InetAddress>> getPendingRanges(String keyspaceName)
+ public PendingRangeMaps getPendingRanges(String keyspaceName)
{
- return getPendingRangesMM(keyspaceName).asMap();
+ return this.pendingRanges.get(keyspaceName);
}
public List<Range<Token>> getPendingRanges(String keyspaceName, InetAddress endpoint)
@@ -748,7 +755,7 @@ public class TokenMetadata
lock.readLock().lock();
try
{
- Multimap<Range<Token>, InetAddress> newPendingRanges = HashMultimap.create();
+ PendingRangeMaps newPendingRanges = new PendingRangeMaps();
if (bootstrapTokens.isEmpty() && leavingEndpoints.isEmpty() && movingEndpoints.isEmpty())
{
@@ -776,7 +783,10 @@ public class TokenMetadata
{
Set<InetAddress> currentEndpoints = ImmutableSet.copyOf(strategy.calculateNaturalEndpoints(range.right, metadata));
Set<InetAddress> newEndpoints = ImmutableSet.copyOf(strategy.calculateNaturalEndpoints(range.right, allLeftMetadata));
- newPendingRanges.putAll(range, Sets.difference(newEndpoints, currentEndpoints));
+ for (InetAddress address : Sets.difference(newEndpoints, currentEndpoints))
+ {
+ newPendingRanges.addPendingRange(range, address);
+ }
}
// At this stage newPendingRanges has been updated according to leave operations. We can
@@ -791,7 +801,9 @@ public class TokenMetadata
allLeftMetadata.updateNormalTokens(tokens, endpoint);
for (Range<Token> range : strategy.getAddressRanges(allLeftMetadata).get(endpoint))
- newPendingRanges.put(range, endpoint);
+ {
+ newPendingRanges.addPendingRange(range, endpoint);
+ }
allLeftMetadata.removeEndpoint(endpoint);
}
@@ -809,7 +821,7 @@ public class TokenMetadata
for (Range<Token> range : strategy.getAddressRanges(allLeftMetadata).get(endpoint))
{
- newPendingRanges.put(range, endpoint);
+ newPendingRanges.addPendingRange(range, endpoint);
}
allLeftMetadata.removeEndpoint(endpoint);
@@ -1046,13 +1058,9 @@ public class TokenMetadata
{
StringBuilder sb = new StringBuilder();
- for (Map.Entry<String, Multimap<Range<Token>, InetAddress>> entry : pendingRanges.entrySet())
+ for (PendingRangeMaps pendingRangeMaps : pendingRanges.values())
{
- for (Map.Entry<Range<Token>, InetAddress> rmap : entry.getValue().entries())
- {
- sb.append(rmap.getValue()).append(':').append(rmap.getKey());
- sb.append(System.getProperty("line.separator"));
- }
+ sb.append(pendingRangeMaps.printPendingRanges());
}
return sb.toString();
@@ -1060,18 +1068,11 @@ public class TokenMetadata
public Collection<InetAddress> pendingEndpointsFor(Token token, String keyspaceName)
{
- Map<Range<Token>, Collection<InetAddress>> ranges = getPendingRanges(keyspaceName);
- if (ranges.isEmpty())
+ PendingRangeMaps pendingRangeMaps = this.pendingRanges.get(keyspaceName);
+ if (pendingRangeMaps == null)
return Collections.emptyList();
- Set<InetAddress> endpoints = new HashSet<>();
- for (Map.Entry<Range<Token>, Collection<InetAddress>> entry : ranges.entrySet())
- {
- if (entry.getKey().contains(token))
- endpoints.addAll(entry.getValue());
- }
-
- return endpoints;
+ return pendingRangeMaps.pendingEndpointsFor(token);
}
/**
http://git-wip-us.apache.org/repos/asf/cassandra/blob/ffc0840d/src/java/org/apache/cassandra/service/StorageService.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/service/StorageService.java b/src/java/org/apache/cassandra/service/StorageService.java
index 24bebae..b74caed 100644
--- a/src/java/org/apache/cassandra/service/StorageService.java
+++ b/src/java/org/apache/cassandra/service/StorageService.java
@@ -1392,7 +1392,7 @@ public class StorageService extends NotificationBroadcasterSupport implements IE
keyspace = Schema.instance.getNonSystemKeyspaces().get(0);
Map<List<String>, List<String>> map = new HashMap<>();
- for (Map.Entry<Range<Token>, Collection<InetAddress>> entry : tokenMetadata.getPendingRanges(keyspace).entrySet())
+ for (Map.Entry<Range<Token>, Collection<InetAddress>> entry : tokenMetadata.getPendingRangesMM(keyspace).asMap().entrySet())
{
List<InetAddress> l = new ArrayList<>(entry.getValue());
map.put(entry.getKey().asList(), stringify(l));