Repository: cassandra
Updated Branches:
  refs/heads/trunk fa592a46b -> e473769fb


Reduce totalBlockFor() for LOCAL_* consistency levels

patch by Nicolas Douillet; reviewed by Aleksey Yeschenko for
CASSANDRA-8058


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

Branch: refs/heads/trunk
Commit: 7497399d0be67b302c852ad92c37e60124b0a1e4
Parents: 0393c30
Author: Nicolas Douillet <nicolas.douil...@gmail.com>
Authored: Sat Oct 11 21:28:04 2014 +0300
Committer: Aleksey Yeschenko <alek...@apache.org>
Committed: Sat Oct 11 21:37:01 2014 +0300

----------------------------------------------------------------------
 CHANGES.txt                                         |  1 +
 .../org/apache/cassandra/db/ConsistencyLevel.java   |  4 ++--
 .../service/DatacenterWriteResponseHandler.java     | 16 +++++++++-------
 3 files changed, 12 insertions(+), 9 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/7497399d/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index b633e48..84bbe75 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 2.0.11:
+ * Reduce totalBlockFor() for LOCAL_* consistency levels (CASSANDRA-8058)
  * Fix merging schemas with re-dropped keyspaces (CASSANDRA-7256)
  * Fix counters in supercolumns during live upgrades from 1.2 (CASSANDRA-7188)
  * Notify DT subscribers when a column family is truncated (CASSANDRA-8088)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/7497399d/src/java/org/apache/cassandra/db/ConsistencyLevel.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/ConsistencyLevel.java 
b/src/java/org/apache/cassandra/db/ConsistencyLevel.java
index 174b80d..3ef3217 100644
--- a/src/java/org/apache/cassandra/db/ConsistencyLevel.java
+++ b/src/java/org/apache/cassandra/db/ConsistencyLevel.java
@@ -145,12 +145,12 @@ public enum ConsistencyLevel
         return isDCLocal;
     }
 
-    private boolean isLocal(InetAddress endpoint)
+    public boolean isLocal(InetAddress endpoint)
     {
         return 
DatabaseDescriptor.getLocalDataCenter().equals(DatabaseDescriptor.getEndpointSnitch().getDatacenter(endpoint));
     }
 
-    private int countLocalEndpoints(Iterable<InetAddress> liveEndpoints)
+    public int countLocalEndpoints(Iterable<InetAddress> liveEndpoints)
     {
         int count = 0;
         for (InetAddress endpoint : liveEndpoints)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/7497399d/src/java/org/apache/cassandra/service/DatacenterWriteResponseHandler.java
----------------------------------------------------------------------
diff --git 
a/src/java/org/apache/cassandra/service/DatacenterWriteResponseHandler.java 
b/src/java/org/apache/cassandra/service/DatacenterWriteResponseHandler.java
index 5530374..74dd488 100644
--- a/src/java/org/apache/cassandra/service/DatacenterWriteResponseHandler.java
+++ b/src/java/org/apache/cassandra/service/DatacenterWriteResponseHandler.java
@@ -20,9 +20,7 @@ package org.apache.cassandra.service;
 import java.net.InetAddress;
 import java.util.Collection;
 
-import org.apache.cassandra.config.DatabaseDescriptor;
 import org.apache.cassandra.db.Keyspace;
-import org.apache.cassandra.locator.IEndpointSnitch;
 import org.apache.cassandra.net.MessageIn;
 import org.apache.cassandra.db.ConsistencyLevel;
 import org.apache.cassandra.db.WriteType;
@@ -32,8 +30,6 @@ import org.apache.cassandra.db.WriteType;
  */
 public class DatacenterWriteResponseHandler extends WriteResponseHandler
 {
-    private static final IEndpointSnitch snitch = 
DatabaseDescriptor.getEndpointSnitch();
-
     public DatacenterWriteResponseHandler(Collection<InetAddress> 
naturalEndpoints,
                                           Collection<InetAddress> 
pendingEndpoints,
                                           ConsistencyLevel consistencyLevel,
@@ -48,10 +44,16 @@ public class DatacenterWriteResponseHandler extends 
WriteResponseHandler
     @Override
     public void response(MessageIn message)
     {
-        if (message == null || 
DatabaseDescriptor.getLocalDataCenter().equals(snitch.getDatacenter(message.from)))
-        {
+        if (message == null || consistencyLevel.isLocal(message.from))
             if (responses.decrementAndGet() == 0)
                 signal();
-        }
+    }
+
+    @Override
+    protected int totalBlockFor()
+    {
+        // during bootstrap, include pending endpoints (only local here) in 
the count
+        // or we may fail the consistency level guarantees (see #833, #8058)
+        return consistencyLevel.blockFor(keyspace) + 
consistencyLevel.countLocalEndpoints(pendingEndpoints);
     }
 }

Reply via email to