Author: jbellis
Date: Thu Oct  7 20:32:52 2010
New Revision: 1005616

URL: http://svn.apache.org/viewvc?rev=1005616&view=rev
Log:
clean up NTS and remove DSS->NTS migration since replicas chosen may be 
different (see #1103).  patch by jbellis; reviewed by Stu Hood for 
CASSANDRA-1593

Modified:
    cassandra/trunk/src/java/org/apache/cassandra/config/KSMetaData.java
    
cassandra/trunk/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
    cassandra/trunk/src/java/org/apache/cassandra/locator/TokenMetadata.java

Modified: cassandra/trunk/src/java/org/apache/cassandra/config/KSMetaData.java
URL: 
http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/config/KSMetaData.java?rev=1005616&r1=1005615&r2=1005616&view=diff
==============================================================================
--- cassandra/trunk/src/java/org/apache/cassandra/config/KSMetaData.java 
(original)
+++ cassandra/trunk/src/java/org/apache/cassandra/config/KSMetaData.java Thu 
Oct  7 20:32:52 2010
@@ -124,7 +124,6 @@ public final class KSMetaData
     public static String convertOldStrategyName(String name)
     {
         return name.replace("RackUnawareStrategy", "SimpleStrategy")
-                   .replace("RackAwareStrategy", "OldNetworkTopologyStrategy")
-                   .replace("DatacenterShardStrategy", 
"NetworkTopologyStrategy");
+                   .replace("RackAwareStrategy", "OldNetworkTopologyStrategy");
     }
 }

Modified: 
cassandra/trunk/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
URL: 
http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java?rev=1005616&r1=1005615&r2=1005616&view=diff
==============================================================================
--- 
cassandra/trunk/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
 (original)
+++ 
cassandra/trunk/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
 Thu Oct  7 20:32:52 2010
@@ -69,59 +69,51 @@ public class NetworkTopologyStrategy ext
 
     public List<InetAddress> calculateNaturalEndpoints(Token searchToken, 
TokenMetadata tokenMetadata)
     {
-        int totalReplicas = getReplicationFactor();
-        Map<String, Integer> remainingReplicas = new HashMap<String, 
Integer>(datacenters);
-        Map<String, Set<String>> dcUsedRacks = new HashMap<String, 
Set<String>>();
-        List<InetAddress> endpoints = new 
ArrayList<InetAddress>(totalReplicas);
-
-        // first pass: only collect replicas on unique racks
-        for (Iterator<Token> iter = 
TokenMetadata.ringIterator(tokenMetadata.sortedTokens(), searchToken);
-             endpoints.size() < totalReplicas && iter.hasNext();)
+        List<InetAddress> endpoints = new 
ArrayList<InetAddress>(getReplicationFactor());
+
+        for (Entry<String, Integer> dcEntry : datacenters.entrySet())
         {
-            Token token = iter.next();
-            InetAddress endpoint = tokenMetadata.getEndpoint(token);
-            String datacenter = snitch.getDatacenter(endpoint);
-            int remaining = remainingReplicas.containsKey(datacenter) ? 
remainingReplicas.get(datacenter) : 0;
-            if (remaining > 0)
+            String dcName = dcEntry.getKey();
+            int dcReplicas = dcEntry.getValue();
+
+            // collect endpoints in this DC
+            TokenMetadata dcTokens = new TokenMetadata();
+            for (Entry<Token, InetAddress> tokenEntry : 
tokenMetadata.entrySet())
             {
-                Set<String> usedRacks = dcUsedRacks.get(datacenter);
-                if (usedRacks == null)
-                {
-                    usedRacks = new HashSet<String>();
-                    dcUsedRacks.put(datacenter, usedRacks);
-                }
+                if (snitch.getDatacenter(tokenEntry.getValue()).equals(dcName))
+                    dcTokens.updateNormalToken(tokenEntry.getKey(), 
tokenEntry.getValue());
+            }
+
+            List<InetAddress> dcEndpoints = new 
ArrayList<InetAddress>(dcReplicas);
+            Set<String> racks = new HashSet<String>();
+            // first pass: only collect replicas on unique racks
+            for (Iterator<Token> iter = 
TokenMetadata.ringIterator(dcTokens.sortedTokens(), searchToken);
+                 dcEndpoints.size() < dcReplicas && iter.hasNext(); )
+            {
+                Token token = iter.next();
+                InetAddress endpoint = dcTokens.getEndpoint(token);
                 String rack = snitch.getRack(endpoint);
-                if (!usedRacks.contains(rack))
+                if (!racks.contains(rack))
                 {
-                    endpoints.add(endpoint);
-                    usedRacks.add(rack);
-                    remainingReplicas.put(datacenter, remaining - 1);
+                    dcEndpoints.add(endpoint);
+                    racks.add(rack);
                 }
             }
-        }
 
-        // second pass: if replica count has not been achieved from unique 
racks, add nodes from the same racks
-        for (Iterator<Token> iter = 
TokenMetadata.ringIterator(tokenMetadata.sortedTokens(), searchToken);
-             endpoints.size() < totalReplicas && iter.hasNext();)
-        {
-            Token token = iter.next();
-            InetAddress endpoint = tokenMetadata.getEndpoint(token);
-            if (endpoints.contains(endpoint))
-                continue;
-
-            String datacenter = snitch.getDatacenter(endpoint);
-            int remaining = remainingReplicas.containsKey(datacenter) ? 
remainingReplicas.get(datacenter) : 0;
-            if (remaining > 0)
+            // second pass: if replica count has not been achieved from unique 
racks, add nodes from duplicate racks
+            for (Iterator<Token> iter = 
TokenMetadata.ringIterator(dcTokens.sortedTokens(), searchToken);
+                 dcEndpoints.size() < dcReplicas && iter.hasNext(); )
             {
-                endpoints.add(endpoint);
-                remainingReplicas.put(datacenter, remaining - 1);
+                Token token = iter.next();
+                InetAddress endpoint = dcTokens.getEndpoint(token);
+                if (!dcEndpoints.contains(endpoint))
+                    dcEndpoints.add(endpoint);
             }
-        }
 
-        for (Map.Entry<String, Integer> entry : remainingReplicas.entrySet())
-        {
-            if (entry.getValue() > 0)
-                throw new IllegalStateException(String.format("datacenter (%s) 
has no more endpoints, (%s) replicas still needed", entry.getKey(), 
entry.getValue()));
+            if (dcEndpoints.size() < dcReplicas)
+                throw new IllegalStateException(String.format("datacenter (%s) 
has no more endpoints, (%s) replicas still needed",
+                                                              dcName, 
dcReplicas - dcEndpoints.size()));
+            endpoints.addAll(dcEndpoints);
         }
 
         return endpoints;

Modified: 
cassandra/trunk/src/java/org/apache/cassandra/locator/TokenMetadata.java
URL: 
http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/locator/TokenMetadata.java?rev=1005616&r1=1005615&r2=1005616&view=diff
==============================================================================
--- cassandra/trunk/src/java/org/apache/cassandra/locator/TokenMetadata.java 
(original)
+++ cassandra/trunk/src/java/org/apache/cassandra/locator/TokenMetadata.java 
Thu Oct  7 20:32:52 2010
@@ -287,6 +287,11 @@ public class TokenMetadata
         }
     }
 
+    public Set<Map.Entry<Token,InetAddress>> entrySet()
+    {
+        return tokenToEndpointMap.entrySet();
+    }
+
     public InetAddress getEndpoint(Token token)
     {
         lock.readLock().lock();


Reply via email to