You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by al...@apache.org on 2013/09/10 17:02:10 UTC

[1/3] git commit: Correctly handle limits in CompositesSearcher

Updated Branches:
  refs/heads/trunk 4f5242cfb -> 36610aaf7


Correctly handle limits in CompositesSearcher

patch by Aleksey Yeschenko; reviewed by Sylvain Lebresne for
CASSANDRA-5975


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

Branch: refs/heads/trunk
Commit: caef32e5d67c3b093de53bf99479cd457169178c
Parents: 30f5e56
Author: Aleksey Yeschenko <al...@apache.org>
Authored: Tue Sep 10 17:41:04 2013 +0300
Committer: Aleksey Yeschenko <al...@apache.org>
Committed: Tue Sep 10 17:41:04 2013 +0300

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../apache/cassandra/db/ColumnFamilyStore.java  |  2 +-
 .../db/index/composites/CompositesSearcher.java | 55 +++++++++++---------
 3 files changed, 33 insertions(+), 25 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/caef32e5/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index a282670..2328bf7 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -8,6 +8,7 @@
  * Make user-defined compaction JMX blocking (CASSANDRA-4952)
  * Fix streaming does not transfer wrapped range (CASSANDRA-5948)
  * Fix loading index summary containing empty key (CASSANDRA-5965)
+ * Correctly handle limits in CompositesSearcher (CASSANDRA-5975)
 
 
 1.2.9

http://git-wip-us.apache.org/repos/asf/cassandra/blob/caef32e5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
index 22b1dd5..745b5ba 100644
--- a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
+++ b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
@@ -1497,7 +1497,7 @@ public class ColumnFamilyStore implements ColumnFamilyStoreMBean
 
         try
         {
-            while (rowIterator.hasNext() && rows.size() < filter.maxRows() && columnsCount < filter.maxColumns())
+            while (rowIterator.hasNext() && matched < filter.maxRows() && columnsCount < filter.maxColumns())
             {
                 // get the raw columns requested, and additional columns for the expressions if necessary
                 Row rawRow = rowIterator.next();

http://git-wip-us.apache.org/repos/asf/cassandra/blob/caef32e5/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java b/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
index 4817a00..1e9d59d 100644
--- a/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
+++ b/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
@@ -21,6 +21,9 @@ import java.io.IOException;
 import java.nio.ByteBuffer;
 import java.util.*;
 
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
 import org.apache.cassandra.db.*;
 import org.apache.cassandra.db.filter.*;
 import org.apache.cassandra.db.index.AbstractSimplePerColumnSecondaryIndex;
@@ -34,8 +37,6 @@ import org.apache.cassandra.dht.Range;
 import org.apache.cassandra.thrift.IndexExpression;
 import org.apache.cassandra.thrift.IndexOperator;
 import org.apache.cassandra.utils.ByteBufferUtil;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
 
 public class CompositesSearcher extends SecondaryIndexSearcher
 {
@@ -111,10 +112,10 @@ public class CompositesSearcher extends SecondaryIndexSearcher
         final CompositeType baseComparator = (CompositeType)baseCfs.getComparator();
         final CompositeType indexComparator = (CompositeType)index.getIndexCfs().getComparator();
 
-        CompositeType.Builder builder = null;
+        final ByteBuffer startPrefix;
         if (startKey.remaining() > 0)
         {
-            builder = indexComparator.builder().add(startKey);
+            CompositeType.Builder builder = indexComparator.builder().add(startKey);
             // For names filter, we have no choice but to query from the beginning of the key. This can be highly inefficient however.
             if (filter.originalFilter() instanceof SliceQueryFilter)
             {
@@ -122,12 +123,17 @@ public class CompositesSearcher extends SecondaryIndexSearcher
                 for (int i = 0; i < Math.min(prefixSize, components.length); ++i)
                     builder.add(components[i]);
             }
+            startPrefix = builder.build();
+        }
+        else
+        {
+            startPrefix = ByteBufferUtil.EMPTY_BYTE_BUFFER;
         }
-        final ByteBuffer startPrefix = startKey.remaining() == 0 ? ByteBufferUtil.EMPTY_BYTE_BUFFER : builder.build();
 
+        final ByteBuffer endPrefix;
         if (endKey.remaining() > 0)
         {
-            builder = indexComparator.builder().add(endKey);
+            CompositeType.Builder builder = indexComparator.builder().add(endKey);
             // For names filter, we have no choice but to query until the end of the key. This can be highly inefficient however.
             if (filter.originalFilter() instanceof SliceQueryFilter)
             {
@@ -135,8 +141,12 @@ public class CompositesSearcher extends SecondaryIndexSearcher
                 for (int i = 0; i < Math.min(prefixSize, components.length); ++i)
                     builder.add(components[i]);
             }
+            endPrefix = builder.buildAsEndOfRange();
+        }
+        else
+        {
+            endPrefix = ByteBufferUtil.EMPTY_BYTE_BUFFER;
         }
-        final ByteBuffer endPrefix = endKey.remaining() == 0 ? ByteBufferUtil.EMPTY_BYTE_BUFFER : builder.buildAsEndOfRange();
 
         // We will need to filter clustering keys based on the user filter. If
         // it is a names filter, we are really interested on the clustering
@@ -150,7 +160,7 @@ public class CompositesSearcher extends SecondaryIndexSearcher
         {
             ByteBuffer first = ((NamesQueryFilter)filter.originalFilter()).columns.iterator().next();
             ByteBuffer[] components = baseComparator.split(first);
-            builder = baseComparator.builder();
+            CompositeType.Builder builder = baseComparator.builder();
             // All all except the last component, since it's the column name
             for (int i = 0; i < components.length - 1; i++)
                 builder.add(components[i]);
@@ -160,11 +170,13 @@ public class CompositesSearcher extends SecondaryIndexSearcher
         return new ColumnFamilyStore.AbstractScanIterator()
         {
             private ByteBuffer lastSeenPrefix = startPrefix;
-            private Deque<IColumn> indexColumns;
+            private ArrayDeque<IColumn> indexColumns;
             private final QueryPath path = new QueryPath(baseCfs.columnFamily);
             private int columnsRead = Integer.MAX_VALUE;
+            private int limit = ((SliceQueryFilter)filter.initialFilter()).count;
+            private int columnsCount = 0;
 
-            private final int meanColumns = Math.max(index.getIndexCfs().getMeanColumns(), 1);
+            private int meanColumns = Math.max(index.getIndexCfs().getMeanColumns(), 1);
             // We shouldn't fetch only 1 row as this provides buggy paging in case the first row doesn't satisfy all clauses
             private final int rowsPerQuery = Math.max(Math.min(filter.maxRows(), filter.maxColumns() / meanColumns), 2);
 
@@ -176,33 +188,27 @@ public class CompositesSearcher extends SecondaryIndexSearcher
             private Row makeReturn(DecoratedKey key, ColumnFamily data)
             {
                 if (data == null)
-                {
                     return endOfData();
-                }
-                else
-                {
-                    assert key != null;
-                    return new Row(key, data);
-                }
+
+                assert key != null;
+                return new Row(key, data);
             }
 
             protected Row computeNext()
             {
                 /*
-                 * Our internal index code is wired toward internal rows. So we need to acumulate all results for a given
+                 * Our internal index code is wired toward internal rows. So we need to accumulate all results for a given
                  * row before returning from this method. Which unfortunately means that this method has to do what
                  * CFS.filter does for KeysIndex.
                  */
                 DecoratedKey currentKey = null;
                 ColumnFamily data = null;
-                int columnsCount = 0;
-                int limit = ((SliceQueryFilter)filter.initialFilter()).count;
 
                 while (true)
                 {
-                    // Did we got more columns that needed to respect the user limit?
-                    // (but we still need to return was fetch already)
-                    if (columnsCount > limit)
+                    // Did we get more columns that needed to respect the user limit?
+                    // (but we still need to return what has been fetched already)
+                    if (columnsCount >= limit)
                         return makeReturn(currentKey, data);
 
                     if (indexColumns == null || indexColumns.isEmpty())
@@ -229,7 +235,7 @@ public class CompositesSearcher extends SecondaryIndexSearcher
 
                         Collection<IColumn> sortedColumns = indexRow.getSortedColumns();
                         columnsRead = sortedColumns.size();
-                        indexColumns = new ArrayDeque(sortedColumns);
+                        indexColumns = new ArrayDeque<IColumn>(sortedColumns);
                         IColumn firstColumn = sortedColumns.iterator().next();
 
                         // Paging is racy, so it is possible the first column of a page is not the last seen one.
@@ -283,6 +289,7 @@ public class CompositesSearcher extends SecondaryIndexSearcher
                             logger.trace("Reached end of assigned scan range");
                             return endOfData();
                         }
+
                         if (!range.contains(dk))
                         {
                             logger.debug("Skipping entry {} outside of assigned scan range", dk.token);


[3/3] git commit: Merge branch 'cassandra-2.0' into trunk

Posted by al...@apache.org.
Merge branch 'cassandra-2.0' into trunk


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

Branch: refs/heads/trunk
Commit: 36610aaf7e99348cae0a989cb28aa8caaa871848
Parents: 4f5242c 096f2bf
Author: Aleksey Yeschenko <al...@apache.org>
Authored: Tue Sep 10 17:58:51 2013 +0300
Committer: Aleksey Yeschenko <al...@apache.org>
Committed: Tue Sep 10 17:58:51 2013 +0300

----------------------------------------------------------------------
 CHANGES.txt                                     |  2 ++
 .../apache/cassandra/db/ColumnFamilyStore.java  |  2 +-
 .../db/index/composites/CompositesSearcher.java | 26 +++++++++-----------
 3 files changed, 14 insertions(+), 16 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/36610aaf/CHANGES.txt
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/36610aaf/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/36610aaf/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
----------------------------------------------------------------------


[2/3] git commit: Merge branch 'cassandra-1.2' into cassandra-2.0

Posted by al...@apache.org.
Merge branch 'cassandra-1.2' into cassandra-2.0

Conflicts:
	CHANGES.txt
	src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java


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

Branch: refs/heads/trunk
Commit: 096f2bf92049ad3d955eb1189526800c8292dd94
Parents: 52678a2 caef32e
Author: Aleksey Yeschenko <al...@apache.org>
Authored: Tue Sep 10 17:58:19 2013 +0300
Committer: Aleksey Yeschenko <al...@apache.org>
Committed: Tue Sep 10 17:58:19 2013 +0300

----------------------------------------------------------------------
 CHANGES.txt                                     |  2 ++
 .../apache/cassandra/db/ColumnFamilyStore.java  |  2 +-
 .../db/index/composites/CompositesSearcher.java | 26 +++++++++-----------
 3 files changed, 14 insertions(+), 16 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/096f2bf9/CHANGES.txt
----------------------------------------------------------------------
diff --cc CHANGES.txt
index abbb4f9,2328bf7..22fa74b
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@@ -23,54 -7,15 +23,56 @@@ Merged from 1.2
   * Allow disabling SlabAllocator (CASSANDRA-5935)
   * Make user-defined compaction JMX blocking (CASSANDRA-4952)
   * Fix streaming does not transfer wrapped range (CASSANDRA-5948)
+  * Fix loading index summary containing empty key (CASSANDRA-5965)
+  * Correctly handle limits in CompositesSearcher (CASSANDRA-5975)
  
  
 -1.2.9
 +2.0.0
 + * Fix thrift validation when inserting into CQL3 tables (CASSANDRA-5138)
 + * Fix periodic memtable flushing behavior with clean memtables (CASSANDRA-5931)
 + * Fix dateOf() function for pre-2.0 timestamp columns (CASSANDRA-5928)
 + * Fix SSTable unintentionally loads BF when opened for batch (CASSANDRA-5938)
 + * Add stream session progress to JMX (CASSANDRA-4757)
 + * Fix NPE during CAS operation (CASSANDRA-5925)
 +Merged from 1.2:
   * Fix getBloomFilterDiskSpaceUsed for AlwaysPresentFilter (CASSANDRA-5900)
 - * migrate 1.1 schema_columnfamilies.key_alias column to key_aliases
 -   (CASSANDRA-5800)
 - * add --migrate option to sstableupgrade and sstablescrub (CASSANDRA-5831)
 + * Don't announce schema version until we've loaded the changes locally
 +   (CASSANDRA-5904)
 + * Fix to support off heap bloom filters size greater than 2 GB (CASSANDRA-5903)
 + * Properly handle parsing huge map and set literals (CASSANDRA-5893)
 +
 +
 +2.0.0-rc2
 + * enable vnodes by default (CASSANDRA-5869)
 + * fix CAS contention timeout (CASSANDRA-5830)
 + * fix HsHa to respect max frame size (CASSANDRA-4573)
 + * Fix (some) 2i on composite components omissions (CASSANDRA-5851)
 + * cqlsh: add DESCRIBE FULL SCHEMA variant (CASSANDRA-5880)
 +Merged from 1.2:
 + * Correctly validate sparse composite cells in scrub (CASSANDRA-5855)
 + * Add KeyCacheHitRate metric to CF metrics (CASSANDRA-5868)
 + * cqlsh: add support for multiline comments (CASSANDRA-5798)
 + * Handle CQL3 SELECT duplicate IN restrictions on clustering columns
 +   (CASSANDRA-5856)
 +
 +
 +2.0.0-rc1
 + * improve DecimalSerializer performance (CASSANDRA-5837)
 + * fix potential spurious wakeup in AsyncOneResponse (CASSANDRA-5690)
 + * fix schema-related trigger issues (CASSANDRA-5774)
 + * Better validation when accessing CQL3 table from thrift (CASSANDRA-5138)
 + * Fix assertion error during repair (CASSANDRA-5801)
 + * Fix range tombstone bug (CASSANDRA-5805)
 + * DC-local CAS (CASSANDRA-5797)
 + * Add a native_protocol_version column to the system.local table (CASSANRDA-5819)
 + * Use index_interval from cassandra.yaml when upgraded (CASSANDRA-5822)
 + * Fix buffer underflow on socket close (CASSANDRA-5792)
 +Merged from 1.2:
 + * Fix reading DeletionTime from 1.1-format sstables (CASSANDRA-5814)
 + * cqlsh: add collections support to COPY (CASSANDRA-5698)
 + * retry important messages for any IOException (CASSANDRA-5804)
 + * Allow empty IN relations in SELECT/UPDATE/DELETE statements (CASSANDRA-5626)
 + * cqlsh: fix crashing on Windows due to libedit detection (CASSANDRA-5812)
   * fix bulk-loading compressed sstables (CASSANDRA-5820)
   * (Hadoop) fix quoting in CqlPagingRecordReader and CqlRecordWriter 
     (CASSANDRA-5824)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/096f2bf9/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/096f2bf9/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
----------------------------------------------------------------------
diff --cc src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
index f9b7b11,1e9d59d..011839e
--- a/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
+++ b/src/java/org/apache/cassandra/db/index/composites/CompositesSearcher.java
@@@ -101,12 -170,15 +101,14 @@@ public class CompositesSearcher extend
          return new ColumnFamilyStore.AbstractScanIterator()
          {
              private ByteBuffer lastSeenPrefix = startPrefix;
 -            private ArrayDeque<IColumn> indexColumns;
 -            private final QueryPath path = new QueryPath(baseCfs.columnFamily);
 +            private Deque<Column> indexColumns;
              private int columnsRead = Integer.MAX_VALUE;
 -            private int limit = ((SliceQueryFilter)filter.initialFilter()).count;
++            private int limit = filter.currentLimit();
+             private int columnsCount = 0;
  
-             private final int meanColumns = Math.max(index.getIndexCfs().getMeanColumns(), 1);
+             private int meanColumns = Math.max(index.getIndexCfs().getMeanColumns(), 1);
              // We shouldn't fetch only 1 row as this provides buggy paging in case the first row doesn't satisfy all clauses
--            private final int rowsPerQuery = Math.max(Math.min(filter.maxRows(), filter.maxColumns() / meanColumns), 2);
++            private int rowsPerQuery = Math.max(Math.min(filter.maxRows(), filter.maxColumns() / meanColumns), 2);
  
              public boolean needsFiltering()
              {
@@@ -138,11 -206,9 +136,9 @@@
  
                  while (true)
                  {
-                     // Did we got more columns that needed to respect the user limit?
-                     // (but we still need to return what was fetch already)
+                     // Did we get more columns that needed to respect the user limit?
+                     // (but we still need to return what has been fetched already)
 -                    if (columnsCount >= limit)
 +                    if (columnsCount > limit)
                          return makeReturn(currentKey, data);
  
                      if (indexColumns == null || indexColumns.isEmpty())
@@@ -162,16 -228,15 +158,16 @@@
                                                                               lastSeenPrefix,
                                                                               endPrefix,
                                                                               false,
 -                                                                             rowsPerQuery);
 +                                                                             rowsPerQuery,
 +                                                                             filter.timestamp);
                          ColumnFamily indexRow = index.getIndexCfs().getColumnFamily(indexFilter);
 -                        if (indexRow == null)
 +                        if (indexRow == null || indexRow.getColumnCount() == 0)
                              return makeReturn(currentKey, data);
  
 -                        Collection<IColumn> sortedColumns = indexRow.getSortedColumns();
 +                        Collection<Column> sortedColumns = indexRow.getSortedColumns();
                          columnsRead = sortedColumns.size();
-                         indexColumns = new ArrayDeque<Column>(sortedColumns);
 -                        indexColumns = new ArrayDeque<IColumn>(sortedColumns);
 -                        IColumn firstColumn = sortedColumns.iterator().next();
++                        indexColumns = new ArrayDeque<>(sortedColumns);
 +                        Column firstColumn = sortedColumns.iterator().next();
  
                          // Paging is racy, so it is possible the first column of a page is not the last seen one.
                          if (lastSeenPrefix != startPrefix && lastSeenPrefix.equals(firstColumn.name()))