You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by ru...@apache.org on 2018/12/05 19:35:08 UTC

cassandra git commit: Auto-expand replication_factor for NetworkTopologyStrategy

Repository: cassandra
Updated Branches:
  refs/heads/trunk eea68a2cf -> 1f19d5f7a


Auto-expand replication_factor for NetworkTopologyStrategy

This re-defines the 'replication_factor' configuration option for
NetworkTopologyStrategy to auto-expand out to the datacenters that the
cluster knows about. This allows users to not worry about their cluster
layout when creating keyspaces. On ALTER this keyword will only ever add
datacenters (it keeps all previous datacenters unless explicitly overridden)

Patch by Joseph Lynch; reviewed by Jon Haddad for CASSANDRA-14303


Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo
Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/1f19d5f7
Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/1f19d5f7
Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/1f19d5f7

Branch: refs/heads/trunk
Commit: 1f19d5f7a243cc4227da923459f5eb2f66066778
Parents: eea68a2
Author: Joseph Lynch <jo...@gmail.com>
Authored: Mon Mar 12 20:37:30 2018 -0700
Committer: Jon Haddad <jo...@jonhaddad.com>
Committed: Wed Dec 5 11:30:55 2018 -0800

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 NEWS.txt                                        |  5 ++
 doc/source/cql/ddl.rst                          | 86 +++++++++++++++---
 .../statements/schema/KeyspaceAttributes.java   |  3 +-
 .../locator/AbstractReplicationStrategy.java    | 34 +++++++
 .../locator/NetworkTopologyStrategy.java        | 46 +++++++++-
 .../cassandra/locator/ReplicationFactor.java    |  7 +-
 .../cassandra/schema/ReplicationParams.java     |  9 +-
 .../org/apache/cassandra/cql3/CQLTester.java    | 23 ++++-
 .../cql3/validation/operations/AlterTest.java   | 93 ++++++++++++++++++++
 .../locator/ReplicationFactorTest.java          | 10 +++
 11 files changed, 295 insertions(+), 22 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index 80bd148..c301e13 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 4.0
+ * Auto-expand replication_factor for NetworkTopologyStrategy (CASSANDRA-14303)
  * Transient Replication: support EACH_QUORUM (CASSANDRA-14727)
  * BufferPool: allocating thread for new chunks should acquire directly (CASSANDRA-14832)
  * Send correct messaging version in internode messaging handshake's third message (CASSANDRA-14896)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/NEWS.txt
----------------------------------------------------------------------
diff --git a/NEWS.txt b/NEWS.txt
index 5cd8542..00c3178 100644
--- a/NEWS.txt
+++ b/NEWS.txt
@@ -105,6 +105,11 @@ New features
    - Faster streaming of entire SSTables using ZeroCopy APIs. If enabled, Cassandra will use stream
      entire SSTables, significantly speeding up transfers. Any streaming related operations will see
      corresponding improvement. See CASSANDRA-14556.
+   - NetworkTopologyStrategy now supports auto-expanding the replication_factor
+     option into all available datacenters at CREATE or ALTER time. For example,
+     specifying replication_factor: 3 translates to three replicas in every
+     datacenter. This auto-expansion will _only add_ datacenters for safety.
+     See CASSANDRA-14303 for more details.
 
 Upgrading
 ---------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/doc/source/cql/ddl.rst
----------------------------------------------------------------------
diff --git a/doc/source/cql/ddl.rst b/doc/source/cql/ddl.rst
index 945e301..afb130e 100644
--- a/doc/source/cql/ddl.rst
+++ b/doc/source/cql/ddl.rst
@@ -73,13 +73,15 @@ A keyspace is created using a ``CREATE KEYSPACE`` statement:
 
 For instance::
 
-    CREATE KEYSPACE Excelsior
-               WITH replication = {'class': 'SimpleStrategy', 'replication_factor' : 3};
+    CREATE KEYSPACE excelsior
+        WITH replication = {'class': 'SimpleStrategy', 'replication_factor' : 3};
 
-    CREATE KEYSPACE Excalibur
-               WITH replication = {'class': 'NetworkTopologyStrategy', 'DC1' : 1, 'DC2' : 3}
-                AND durable_writes = false;
+    CREATE KEYSPACE excalibur
+        WITH replication = {'class': 'NetworkTopologyStrategy', 'DC1' : 1, 'DC2' : 3}
+        AND durable_writes = false;
 
+Attempting to create a keyspace that already exists will return an error unless the ``IF NOT EXISTS`` option is used. If
+it is used, the statement will be a no-op if the keyspace already exists.
 
 The supported ``options`` are:
 
@@ -96,14 +98,72 @@ The ``replication`` property is mandatory and must at least contains the ``'clas
 :ref:`replication strategy <replication-strategy>` class to use. The rest of the sub-options depends on what replication
 strategy is used. By default, Cassandra support the following ``'class'``:
 
-- ``'SimpleStrategy'``: A simple strategy that defines a replication factor for the whole cluster. The only sub-options
-  supported is ``'replication_factor'`` to define that replication factor and is mandatory.
-- ``'NetworkTopologyStrategy'``: A replication strategy that allows to set the replication factor independently for
-  each data-center. The rest of the sub-options are key-value pairs where a key is a data-center name and its value is
-  the associated replication factor.
+``SimpleStrategy``
+""""""""""""""""""
 
-Attempting to create a keyspace that already exists will return an error unless the ``IF NOT EXISTS`` option is used. If
-it is used, the statement will be a no-op if the keyspace already exists.
+A simple strategy that defines a replication factor for data to be spread
+across the entire cluster. This is generally not a wise choice for production
+because it does not respect datacenter layouts and can lead to wildly varying
+query latency. For a production ready strategy, see
+``NetworkTopologyStrategy``. ``SimpleStrategy`` supports a single mandatory argument:
+
+========================= ====== ======= =============================================
+sub-option                 type   since   description
+========================= ====== ======= =============================================
+``'replication_factor'``   int    all     The number of replicas to store per range
+========================= ====== ======= =============================================
+
+``NetworkTopologyStrategy``
+"""""""""""""""""""""""""""
+
+A production ready replication strategy that allows to set the replication
+factor independently for each data-center. The rest of the sub-options are
+key-value pairs where a key is a data-center name and its value is the
+associated replication factor. Options:
+
+===================================== ====== ====== =============================================
+sub-option                             type   since  description
+===================================== ====== ====== =============================================
+``'<datacenter>'``                     int    all    The number of replicas to store per range in
+                                                     the provided datacenter.
+``'replication_factor'``               int    4.0    The number of replicas to use as a default
+                                                     per datacenter if not specifically provided.
+                                                     Note that this always defers to existing
+                                                     definitions or explicit datacenter settings.
+                                                     For example, to have three replicas per
+                                                     datacenter, supply this with a value of 3.
+===================================== ====== ====== =============================================
+
+Note that when ``ALTER`` ing keyspaces and supplying ``replication_factor``,
+auto-expansion will only *add* new datacenters for safety, it will not alter
+existing datacenters or remove any even if they are no longer in the cluster.
+If you want to remove datacenters while still supplying ``replication_factor``,
+explicitly zero out the datacenter you want to have zero replicas.
+
+An example of auto-expanding datacenters with two datacenters: ``DC1`` and ``DC2``::
+
+    CREATE KEYSPACE excalibur
+        WITH replication = {'class': 'NetworkTopologyStrategy', 'replication_factor' : 3}
+
+    DESCRIBE KEYSPACE excalibur
+        CREATE KEYSPACE excalibur WITH replication = {'class': 'NetworkTopologyStrategy', 'DC1': '3', 'DC2': '3'} AND durable_writes = true;
+
+
+An example of auto-expanding and overriding a datacenter::
+
+    CREATE KEYSPACE excalibur
+        WITH replication = {'class': 'NetworkTopologyStrategy', 'replication_factor' : 3, 'DC2': 2}
+
+    DESCRIBE KEYSPACE excalibur
+        CREATE KEYSPACE excalibur WITH replication = {'class': 'NetworkTopologyStrategy', 'DC1': '3', 'DC2': '2'} AND durable_writes = true;
+
+An example that excludes a datacenter while using ``replication_factor``::
+
+    CREATE KEYSPACE excalibur
+        WITH replication = {'class': 'NetworkTopologyStrategy', 'replication_factor' : 3, 'DC2': 0} ;
+
+    DESCRIBE KEYSPACE excalibur
+        CREATE KEYSPACE excalibur WITH replication = {'class': 'NetworkTopologyStrategy', 'DC1': '3'} AND durable_writes = true;
 
 If :ref:`transient replication <transient-replication>` has been enabled, transient replicas can be configured for both
 SimpleStrategy and NetworkTopologyStrategy by defining replication factors in the format ``'<total_replicas>/<transient_replicas>'``
@@ -139,7 +199,7 @@ An ``ALTER KEYSPACE`` statement allows to modify the options of a keyspace:
 For instance::
 
     ALTER KEYSPACE Excelsior
-              WITH replication = {'class': 'SimpleStrategy', 'replication_factor' : 4};
+        WITH replication = {'class': 'SimpleStrategy', 'replication_factor' : 4};
 
 The supported options are the same than for :ref:`creating a keyspace <create-keyspace-statement>`.
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/src/java/org/apache/cassandra/cql3/statements/schema/KeyspaceAttributes.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/cql3/statements/schema/KeyspaceAttributes.java b/src/java/org/apache/cassandra/cql3/statements/schema/KeyspaceAttributes.java
index 06c16a9..42fcaf4 100644
--- a/src/java/org/apache/cassandra/cql3/statements/schema/KeyspaceAttributes.java
+++ b/src/java/org/apache/cassandra/cql3/statements/schema/KeyspaceAttributes.java
@@ -72,9 +72,10 @@ public final class KeyspaceAttributes extends PropertyDefinitions
     KeyspaceParams asAlteredKeyspaceParams(KeyspaceParams previous)
     {
         boolean durableWrites = getBoolean(Option.DURABLE_WRITES.toString(), previous.durableWrites);
+        Map<String, String> previousOptions = previous.replication.options;
         ReplicationParams replication = getReplicationStrategyClass() == null
                                       ? previous.replication
-                                      : ReplicationParams.fromMap(getAllReplicationOptions());
+                                      : ReplicationParams.fromMapWithDefaults(getAllReplicationOptions(), previousOptions);
         return new KeyspaceParams(durableWrites, replication);
     }
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/src/java/org/apache/cassandra/locator/AbstractReplicationStrategy.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/locator/AbstractReplicationStrategy.java b/src/java/org/apache/cassandra/locator/AbstractReplicationStrategy.java
index deb43c6..874097d 100644
--- a/src/java/org/apache/cassandra/locator/AbstractReplicationStrategy.java
+++ b/src/java/org/apache/cassandra/locator/AbstractReplicationStrategy.java
@@ -19,6 +19,7 @@ package org.apache.cassandra.locator;
 
 import java.lang.reflect.Constructor;
 import java.lang.reflect.InvocationTargetException;
+import java.lang.reflect.Method;
 import java.util.*;
 
 import com.google.common.annotations.VisibleForTesting;
@@ -367,6 +368,39 @@ public abstract class AbstractReplicationStrategy
         return strategy;
     }
 
+    /**
+     * Before constructing the ARS we first give it a chance to prepare the options map in any way it
+     * would like to. For example datacenter auto-expansion or other templating to make the user interface
+     * more usable. Note that this may mutate the passed strategyOptions Map.
+     *
+     * We do this prior to the construction of the strategyClass itself because at that point the option
+     * map is already immutable and comes from {@link org.apache.cassandra.schema.ReplicationParams}
+     * (and should probably stay that way so we don't start having bugs related to ReplicationParams being mutable).
+     * Instead ARS classes get a static hook here via the prepareOptions(Map, Map) method to mutate the user input
+     * before it becomes an immutable part of the ReplicationParams.
+     *
+     * @param strategyClass The class to call prepareOptions on
+     * @param strategyOptions The proposed strategy options that will be potentially mutated by the prepareOptions
+     *                        method.
+     * @param previousStrategyOptions In the case of an ALTER statement, the previous strategy options of this class.
+     *                                This map cannot be mutated.
+     */
+    public static void prepareReplicationStrategyOptions(Class<? extends AbstractReplicationStrategy> strategyClass,
+                                                         Map<String, String> strategyOptions,
+                                                         Map<String, String> previousStrategyOptions)
+    {
+        try
+        {
+            Method method = strategyClass.getDeclaredMethod("prepareOptions", Map.class, Map.class);
+            method.invoke(null, strategyOptions, previousStrategyOptions);
+        }
+        catch (NoSuchMethodException | IllegalAccessException | InvocationTargetException ign)
+        {
+            // If the subclass doesn't specify a prepareOptions method, then that means that it
+            // doesn't want to do anything to the options. So do nothing on reflection related exceptions.
+        }
+    }
+
     public static void validateReplicationStrategy(String keyspaceName,
                                                    Class<? extends AbstractReplicationStrategy> strategyClass,
                                                    TokenMetadata tokenMetadata,

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java b/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
index 713a75e..c9f24e0 100644
--- a/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
+++ b/src/java/org/apache/cassandra/locator/NetworkTopologyStrategy.java
@@ -54,6 +54,7 @@ public class NetworkTopologyStrategy extends AbstractReplicationStrategy
     private final Map<String, ReplicationFactor> datacenters;
     private final ReplicationFactor aggregateRf;
     private static final Logger logger = LoggerFactory.getLogger(NetworkTopologyStrategy.class);
+    private static final String REPLICATION_FACTOR = "replication_factor";
 
     public NetworkTopologyStrategy(String keyspaceName, TokenMetadata tokenMetadata, IEndpointSnitch snitch, Map<String, String> configOptions) throws ConfigurationException
     {
@@ -67,8 +68,9 @@ public class NetworkTopologyStrategy extends AbstractReplicationStrategy
             for (Entry<String, String> entry : configOptions.entrySet())
             {
                 String dc = entry.getKey();
-                if (dc.equalsIgnoreCase("replication_factor"))
-                    throw new ConfigurationException("replication_factor is an option for SimpleStrategy, not NetworkTopologyStrategy");
+                // prepareOptions should have transformed any "replication_factor" options by now
+                if (dc.equalsIgnoreCase(REPLICATION_FACTOR))
+                    throw new ConfigurationException(REPLICATION_FACTOR + " should not appear as an option at construction time for NetworkTopologyStrategy");
                 ReplicationFactor rf = ReplicationFactor.fromString(entry.getValue());
                 replicas += rf.allReplicas;
                 trans += rf.transientReplicas();
@@ -241,6 +243,41 @@ public class NetworkTopologyStrategy extends AbstractReplicationStrategy
         return Datacenters.getValidDatacenters();
     }
 
+    /**
+     * Support datacenter auto-expansion for CASSANDRA-14303. This hook allows us to safely auto-expand
+     * the "replication_factor" options out into the known datacenters. It is called via reflection from
+     * {@link AbstractReplicationStrategy#prepareReplicationStrategyOptions(Class, Map, Map)}.
+     *
+     * @param options The proposed strategy options that will be potentially mutated
+     * @param previousOptions Any previous strategy options in the case of an ALTER statement
+     */
+    protected static void prepareOptions(Map<String, String> options, Map<String, String> previousOptions)
+    {
+        String replication = options.remove(REPLICATION_FACTOR);
+
+        if (replication == null && options.size() == 0)
+        {
+            // Support direct alters from SimpleStrategy to NTS
+            replication = previousOptions.get(REPLICATION_FACTOR);
+        }
+        else if (replication != null)
+        {
+            // When datacenter auto-expansion occurs in e.g. an ALTER statement (meaning that the previousOptions
+            // map is not empty) we choose not to alter existing datacenter replication levels for safety.
+            previousOptions.entrySet().stream()
+                           .filter(e -> !e.getKey().equals(REPLICATION_FACTOR)) // SimpleStrategy conversions
+                           .forEach(e -> options.putIfAbsent(e.getKey(), e.getValue()));
+        }
+
+        if (replication != null) {
+            ReplicationFactor defaultReplicas = ReplicationFactor.fromString(replication);
+            Datacenters.getValidDatacenters()
+                       .forEach(dc -> options.putIfAbsent(dc, defaultReplicas.toParseableString()));
+        }
+
+        options.values().removeAll(Collections.singleton("0"));
+    }
+
     protected void validateExpectedOptions() throws ConfigurationException
     {
         // Do not accept query with no data centers specified.
@@ -257,8 +294,9 @@ public class NetworkTopologyStrategy extends AbstractReplicationStrategy
     {
         for (Entry<String, String> e : this.configOptions.entrySet())
         {
-            if (e.getKey().equalsIgnoreCase("replication_factor"))
-                throw new ConfigurationException("replication_factor is an option for SimpleStrategy, not NetworkTopologyStrategy");
+            // prepareOptions should have transformed any "replication_factor" by now
+            if (e.getKey().equalsIgnoreCase(REPLICATION_FACTOR))
+                throw new ConfigurationException(REPLICATION_FACTOR + " should not appear as an option to NetworkTopologyStrategy");
             validateReplicationFactor(e.getValue());
         }
     }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/src/java/org/apache/cassandra/locator/ReplicationFactor.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/locator/ReplicationFactor.java b/src/java/org/apache/cassandra/locator/ReplicationFactor.java
index c0ed31f..91ce770 100644
--- a/src/java/org/apache/cassandra/locator/ReplicationFactor.java
+++ b/src/java/org/apache/cassandra/locator/ReplicationFactor.java
@@ -122,9 +122,14 @@ public class ReplicationFactor
         }
     }
 
+    public String toParseableString()
+    {
+        return String.valueOf(allReplicas) + (hasTransientReplicas() ? "/" + transientReplicas() : "");
+    }
+
     @Override
     public String toString()
     {
-        return "rf(" + allReplicas + (hasTransientReplicas() ? '/' + transientReplicas() : "") + ')';
+        return "rf(" + toParseableString() + ')';
     }
 }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/src/java/org/apache/cassandra/schema/ReplicationParams.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/schema/ReplicationParams.java b/src/java/org/apache/cassandra/schema/ReplicationParams.java
index 21e19d6..5f744f8 100644
--- a/src/java/org/apache/cassandra/schema/ReplicationParams.java
+++ b/src/java/org/apache/cassandra/schema/ReplicationParams.java
@@ -77,11 +77,18 @@ public final class ReplicationParams
         AbstractReplicationStrategy.validateReplicationStrategy(name, klass, tmd, eps, options);
     }
 
-    public static ReplicationParams fromMap(Map<String, String> map)
+    public static ReplicationParams fromMap(Map<String, String> map) {
+        return fromMapWithDefaults(map, new HashMap<>());
+    }
+
+    public static ReplicationParams fromMapWithDefaults(Map<String, String> map, Map<String, String> defaults)
     {
         Map<String, String> options = new HashMap<>(map);
         String className = options.remove(CLASS);
+
         Class<? extends AbstractReplicationStrategy> klass = AbstractReplicationStrategy.getClass(className);
+        AbstractReplicationStrategy.prepareReplicationStrategyOptions(klass, options, defaults);
+
         return new ReplicationParams(klass, options);
     }
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/test/unit/org/apache/cassandra/cql3/CQLTester.java
----------------------------------------------------------------------
diff --git a/test/unit/org/apache/cassandra/cql3/CQLTester.java b/test/unit/org/apache/cassandra/cql3/CQLTester.java
index e6b0e29..1eb56bc 100644
--- a/test/unit/org/apache/cassandra/cql3/CQLTester.java
+++ b/test/unit/org/apache/cassandra/cql3/CQLTester.java
@@ -23,6 +23,7 @@ import java.math.BigDecimal;
 import java.math.BigInteger;
 import java.net.InetAddress;
 import java.net.ServerSocket;
+import java.net.UnknownHostException;
 import java.nio.ByteBuffer;
 import java.util.*;
 import java.util.concurrent.CountDownLatch;
@@ -50,6 +51,7 @@ import org.apache.cassandra.index.SecondaryIndexManager;
 import org.apache.cassandra.config.EncryptionOptions;
 import org.apache.cassandra.locator.InetAddressAndPort;
 import org.apache.cassandra.locator.Replica;
+import org.apache.cassandra.locator.TokenMetadata;
 import org.apache.cassandra.schema.*;
 import org.apache.cassandra.config.DatabaseDescriptor;
 import org.apache.cassandra.cql3.functions.FunctionName;
@@ -93,11 +95,13 @@ public abstract class CQLTester
     private static final AtomicInteger seqNumber = new AtomicInteger();
     protected static final ByteBuffer TOO_BIG = ByteBuffer.allocate(FBUtilities.MAX_UNSIGNED_SHORT + 1024);
     public static final String DATA_CENTER = "datacenter1";
+    public static final String DATA_CENTER_REMOTE = "datacenter2";
     public static final String RACK1 = "rack1";
 
     private static org.apache.cassandra.transport.Server server;
     protected static final int nativePort;
     protected static final InetAddress nativeAddr;
+    protected static final Set<InetAddressAndPort> remoteAddrs = new HashSet<>();
     private static final Map<ProtocolVersion, Cluster> clusters = new HashMap<>();
     protected static final Map<ProtocolVersion, Session> sessions = new HashMap<>();
 
@@ -148,7 +152,11 @@ public abstract class CQLTester
         DatabaseDescriptor.setEndpointSnitch(new AbstractEndpointSnitch()
         {
             @Override public String getRack(InetAddressAndPort endpoint) { return RACK1; }
-            @Override public String getDatacenter(InetAddressAndPort endpoint) { return DATA_CENTER; }
+            @Override public String getDatacenter(InetAddressAndPort endpoint) {
+                if (remoteAddrs.contains(endpoint))
+                    return DATA_CENTER_REMOTE;
+                return DATA_CENTER;
+            }
             @Override public int compareEndpoints(InetAddressAndPort target, Replica a1, Replica a2) { return 0; }
         });
 
@@ -203,6 +211,15 @@ public abstract class CQLTester
             throw new RuntimeException(e);
         }
 
+        try {
+            remoteAddrs.add(InetAddressAndPort.getByName("127.0.0.4"));
+        }
+        catch (UnknownHostException e)
+        {
+            logger.error("Failed to lookup host");
+            throw new RuntimeException(e);
+        }
+
         Thread.setDefaultUncaughtExceptionHandler(new Thread.UncaughtExceptionHandler()
         {
             public void uncaughtException(Thread t, Throwable e)
@@ -276,7 +293,6 @@ public abstract class CQLTester
     {
         if (ROW_CACHE_SIZE_IN_MB > 0)
             DatabaseDescriptor.setRowCacheSizeInMB(ROW_CACHE_SIZE_IN_MB);
-
         StorageService.instance.setPartitionerUnsafe(Murmur3Partitioner.instance);
 
         // Once per-JVM is enough
@@ -298,6 +314,9 @@ public abstract class CQLTester
         // statements are not cached but re-prepared every time). So we clear the cache between test files to avoid accumulating too much.
         if (reusePrepared)
             QueryProcessor.clearInternalStatementsCache();
+
+        TokenMetadata metadata = StorageService.instance.getTokenMetadata();
+        metadata.clearUnsafe();
     }
 
     @Before

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
----------------------------------------------------------------------
diff --git a/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java b/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
index db83eb4..ea78f88 100644
--- a/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
+++ b/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
@@ -17,9 +17,14 @@
  */
 package org.apache.cassandra.cql3.validation.operations;
 
+import java.util.UUID;
+
 import org.junit.Assert;
 import org.junit.Test;
 
+import org.apache.cassandra.dht.OrderPreservingPartitioner;
+import org.apache.cassandra.locator.InetAddressAndPort;
+import org.apache.cassandra.locator.TokenMetadata;
 import org.apache.cassandra.schema.SchemaConstants;
 import org.apache.cassandra.cql3.CQLTester;
 import org.apache.cassandra.db.ColumnFamilyStore;
@@ -27,6 +32,8 @@ import org.apache.cassandra.db.Keyspace;
 import org.apache.cassandra.exceptions.ConfigurationException;
 import org.apache.cassandra.exceptions.SyntaxException;
 import org.apache.cassandra.schema.SchemaKeyspace;
+import org.apache.cassandra.service.StorageService;
+import org.apache.cassandra.utils.FBUtilities;
 
 import static java.lang.String.format;
 import static org.junit.Assert.assertEquals;
@@ -244,6 +251,92 @@ public class AlterTest extends CQLTester
                                   "max_threshold", "32")));
     }
 
+    @Test
+    public void testCreateAlterNetworkTopologyWithDefaults() throws Throwable
+    {
+        TokenMetadata metadata = StorageService.instance.getTokenMetadata();
+        metadata.clearUnsafe();
+        InetAddressAndPort local = FBUtilities.getBroadcastAddressAndPort();
+        InetAddressAndPort remote = InetAddressAndPort.getByName("127.0.0.4");
+        metadata.updateHostId(UUID.randomUUID(), local);
+        metadata.updateNormalToken(new OrderPreservingPartitioner.StringToken("A"), local);
+        metadata.updateHostId(UUID.randomUUID(), remote);
+        metadata.updateNormalToken(new OrderPreservingPartitioner.StringToken("B"), remote);
+
+        // With two datacenters we should respect anything passed in as a manual override
+        String ks1 = createKeyspace("CREATE KEYSPACE %s WITH replication={ 'class' : 'NetworkTopologyStrategy', 'replication_factor' : 1, '" + DATA_CENTER_REMOTE + "': 3}");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.NetworkTopologyStrategy", DATA_CENTER, "1", DATA_CENTER_REMOTE, "3")));
+
+        // Should be able to remove data centers
+        schemaChange("ALTER KEYSPACE " + ks1 + " WITH replication = { 'class' : 'NetworkTopologyStrategy', '" + DATA_CENTER + "' : 0, '" + DATA_CENTER_REMOTE + "': 3 }");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.NetworkTopologyStrategy", DATA_CENTER_REMOTE, "3")));
+
+        // The auto-expansion should not change existing replication counts; do not let the user shoot themselves in the foot
+        schemaChange("ALTER KEYSPACE " + ks1 + " WITH replication = { 'class' : 'NetworkTopologyStrategy', 'replication_factor' : 1 } AND durable_writes=True");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.NetworkTopologyStrategy", DATA_CENTER, "1", DATA_CENTER_REMOTE, "3")));
+
+        // The keyspace should be fully functional
+        execute("USE " + ks1);
+
+        assertInvalidThrow(ConfigurationException.class, "CREATE TABLE tbl1 (a int PRIMARY KEY, b int) WITH compaction = { 'min_threshold' : 4 }");
+
+        execute("CREATE TABLE tbl1 (a int PRIMARY KEY, b int) WITH compaction = { 'class' : 'SizeTieredCompactionStrategy', 'min_threshold' : 7 }");
+
+        assertRows(execute("SELECT table_name, compaction FROM system_schema.tables WHERE keyspace_name='" + ks1 + "'"),
+                   row("tbl1", map("class", "org.apache.cassandra.db.compaction.SizeTieredCompactionStrategy",
+                                  "min_threshold", "7",
+                                  "max_threshold", "32")));
+    }
+
+    @Test
+    public void testCreateSimpleAlterNTSDefaults() throws Throwable
+    {
+        TokenMetadata metadata = StorageService.instance.getTokenMetadata();
+        metadata.clearUnsafe();
+        InetAddressAndPort local = FBUtilities.getBroadcastAddressAndPort();
+        InetAddressAndPort remote = InetAddressAndPort.getByName("127.0.0.4");
+        metadata.updateHostId(UUID.randomUUID(), local);
+        metadata.updateNormalToken(new OrderPreservingPartitioner.StringToken("A"), local);
+        metadata.updateHostId(UUID.randomUUID(), remote);
+        metadata.updateNormalToken(new OrderPreservingPartitioner.StringToken("B"), remote);
+
+        // Let's create a keyspace first with SimpleStrategy
+        String ks1 = createKeyspace("CREATE KEYSPACE %s WITH replication={ 'class' : 'SimpleStrategy', 'replication_factor' : 2}");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "2")));
+
+        // Now we should be able to ALTER to NetworkTopologyStrategy directly from SimpleStrategy without supplying replication_factor
+        schemaChange("ALTER KEYSPACE " + ks1 + " WITH replication = { 'class' : 'NetworkTopologyStrategy'}");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.NetworkTopologyStrategy", DATA_CENTER, "2", DATA_CENTER_REMOTE, "2")));
+
+        schemaChange("ALTER KEYSPACE " + ks1 + " WITH replication = { 'class' : 'SimpleStrategy', 'replication_factor' : 3}");
+        schemaChange("ALTER KEYSPACE " + ks1 + " WITH replication = { 'class' : 'NetworkTopologyStrategy', 'replication_factor': 2}");
+
+        assertRowsIgnoringOrderAndExtra(execute("SELECT keyspace_name, durable_writes, replication FROM system_schema.keyspaces"),
+                                        row(KEYSPACE, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(KEYSPACE_PER_TEST, true, map("class", "org.apache.cassandra.locator.SimpleStrategy", "replication_factor", "1")),
+                                        row(ks1, true, map("class", "org.apache.cassandra.locator.NetworkTopologyStrategy", DATA_CENTER, "2", DATA_CENTER_REMOTE, "2")));
+    }
+
     /**
      * Test {@link ConfigurationException} thrown on alter keyspace to no DC option in replication configuration.
      */

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1f19d5f7/test/unit/org/apache/cassandra/locator/ReplicationFactorTest.java
----------------------------------------------------------------------
diff --git a/test/unit/org/apache/cassandra/locator/ReplicationFactorTest.java b/test/unit/org/apache/cassandra/locator/ReplicationFactorTest.java
index a0427db..7d85b44 100644
--- a/test/unit/org/apache/cassandra/locator/ReplicationFactorTest.java
+++ b/test/unit/org/apache/cassandra/locator/ReplicationFactorTest.java
@@ -70,4 +70,14 @@ public class ReplicationFactorTest
         assertRfParseFailure("3/3");
         assertRfParseFailure("3/4");
     }
+
+    @Test
+    public void roundTripParseTest()
+    {
+        String input = "3";
+        Assert.assertEquals(input, ReplicationFactor.fromString(input).toParseableString());
+
+        String transientInput = "3/1";
+        Assert.assertEquals(transientInput, ReplicationFactor.fromString(transientInput).toParseableString());
+    }
 }


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