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 2017/03/27 10:04:40 UTC

[1/6] cassandra git commit: Dropping column results in "corrupt" SSTable

Repository: cassandra
Updated Branches:
  refs/heads/cassandra-3.0 631162271 -> 5262bb17b
  refs/heads/cassandra-3.11 ee7023e32 -> c0ac928d9
  refs/heads/trunk 23cd27fcf -> 3dabeeaa2


Dropping column results in "corrupt" SSTable

patch by Sylvain Lebresne; reviewed by Alex Petrov for CASSANDRA-13337


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

Branch: refs/heads/cassandra-3.0
Commit: 5262bb17b46fc8c02f9f836ddf9317d0de2698cd
Parents: 6311622
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 20 15:49:27 2017 +0100
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 11:58:32 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  3 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 144 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index 2c5573a..0b1bb01 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 3.0.13
+ * Dropping column results in "corrupt" SSTable (CASSANDRA-13337)
  * Bugs handling range tombstones in the sstable iterators (CASSANDRA-13340)
  * Fix CONTAINS filtering for null collections (CASSANDRA-13246)
  * Applying: Use a unique metric reservoir per test run when using Cassandra-wide metrics residing in MBeans (CASSANDRA-13216)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
index 9bcca48..fa337c0 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
@@ -123,20 +123,27 @@ public class SSTableIterator extends AbstractSSTableIterator
         {
             assert deserializer != null;
 
-            // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
-            // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
-            // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
-            // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
-            // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
-            // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
-            // an open marker" code in hasNextInterna
-            if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
-                return null;
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
+                // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
+                // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
+                // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
+                // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
+                // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
+                // an open marker" code in hasNextInterna
+                if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
+                    return null;
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
 
         protected boolean hasNextInternal() throws IOException
@@ -256,24 +263,31 @@ public class SSTableIterator extends AbstractSSTableIterator
         @Override
         protected Unfiltered computeNext() throws IOException
         {
-            // Our previous read might have made us cross an index block boundary. If so, update our informations.
-            // If we read from the beginning of the partition, this is also what will initialize the index state.
-            indexState.updateBlock();
-
-            // Return the next unfiltered unless we've reached the end, or we're beyond our slice
-            // end (note that unless we're on the last block for the slice, there is no point
-            // in checking the slice end).
-            if (indexState.isDone()
-                || indexState.currentBlockIdx() > lastBlockIdx
-                || !deserializer.hasNext()
-                || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
-                return null;
-
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // Our previous read might have made us cross an index block boundary. If so, update our informations.
+                // If we read from the beginning of the partition, this is also what will initialize the index state.
+                indexState.updateBlock();
+
+                // Return the next unfiltered unless we've reached the end, or we're beyond our slice
+                // end (note that unless we're on the last block for the slice, there is no point
+                // in checking the slice end).
+                if (indexState.isDone()
+                    || indexState.currentBlockIdx() > lastBlockIdx
+                    || !deserializer.hasNext()
+                    || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
+                    return null;
+
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
index 49dc82a..4bb7fe8 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
@@ -213,7 +213,8 @@ public class SSTableReversedIterator extends AbstractSSTableIterator
                    && !stopReadingDisk())
             {
                 Unfiltered unfiltered = deserializer.readNext();
-                if (!isFirst || includeFirst)
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (!unfiltered.isEmpty() && (!isFirst || includeFirst))
                     buffer.add(unfiltered);
 
                 isFirst = false;

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
index 1cd5fb4..c4c9f7f 100644
--- a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
+++ b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
@@ -49,6 +49,12 @@ public interface RangeTombstoneMarker extends Unfiltered
 
     public RangeTombstoneMarker copy(AbstractAllocator allocator);
 
+    default public boolean isEmpty()
+    {
+        // There is no such thing as an empty marker
+        return false;
+    }
+
     /**
      * Utility class to help merging range tombstone markers coming from multiple inputs (UnfilteredRowIterators).
      * <p>

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/Unfiltered.java b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
index 9d96137..9511eeb 100644
--- a/src/java/org/apache/cassandra/db/rows/Unfiltered.java
+++ b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
@@ -55,6 +55,8 @@ public interface Unfiltered extends Clusterable
      */
     public void validateData(CFMetaData metadata);
 
+    public boolean isEmpty();
+
     public String toString(CFMetaData metadata);
     public String toString(CFMetaData metadata, boolean fullDetails);
     public String toString(CFMetaData metadata, boolean includeClusterKeys, boolean fullDetails);

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
index dc6f187..bdc8388 100644
--- a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
+++ b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
@@ -350,9 +350,44 @@ public class UnfilteredSerializer
         return 1;
     }
 
+    /**
+     * Deserialize an {@link Unfiltered} from the provided input.
+     *
+     * @param in the input from which to deserialize.
+     * @param header serialization header corresponding to the serialized data.
+     * @param helper the helper to use for deserialization.
+     * @param builder a row builder, passed here so we don't allocate a new one for every new row.
+     * @return the deserialized {@link Unfiltered} or {@code null} if we've read the end of a partition. This method is
+     * guaranteed to never return empty rows.
+     */
     public Unfiltered deserialize(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
     throws IOException
     {
+        while (true)
+        {
+            Unfiltered unfiltered = deserializeOne(in, header, helper, builder);
+            if (unfiltered == null)
+                return null;
+
+            // Skip empty rows, see deserializeOne javadoc
+            if (!unfiltered.isEmpty())
+                return unfiltered;
+        }
+    }
+
+    /**
+     * Deserialize a single {@link Unfiltered} from the provided input.
+     * <p>
+     * <b>WARNING:</b> this can return an empty row because it's possible there is a row serialized, but that row only
+     * contains data for dropped columns, see CASSANDRA-13337. But as most code expect rows to not be empty, this isn't
+     * meant to be exposed publicly.
+     *
+     * But as {@link UnfilteredRowIterator} should not return empty
+     * rows, this mean consumer of this method should make sure to skip said empty rows.
+     */
+    private Unfiltered deserializeOne(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
+    throws IOException
+    {
         // It wouldn't be wrong per-se to use an unsorted builder, but it would be inefficient so make sure we don't do it by mistake
         assert builder.isSorted();
 
@@ -374,13 +409,7 @@ public class UnfilteredSerializer
                 throw new IOException("Corrupt flags value for unfiltered partition (isStatic flag set): " + flags);
 
             builder.newRow(Clustering.serializer.deserialize(in, helper.version, header.clusteringTypes()));
-            Row row = deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
-            // we do not write empty rows because Rows.collectStats(), called by BTW.applyToRow(), asserts that rows are not empty
-            // if we don't throw here, then later the very same assertion in Rows.collectStats() will fail compactions
-            // see BlackListingCompactionsTest and CASSANDRA-9530 for details
-            if (row.isEmpty())
-                throw new IOException("Corrupt empty row found in unfiltered partition");
-            return row;
+            return deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
         }
     }
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/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 245be30..c48ffe5 100644
--- a/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
+++ b/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
@@ -355,4 +355,55 @@ public class AlterTest extends CQLTester
             assertEquals(errorMsg, e.getMessage());
         }
     }
+
+    /**
+     * Test for CASSANDRA-13337. Checks that dropping a column when a sstable contains only data for that column
+     * works properly.
+     */
+    @Test
+    public void testAlterDropEmptySSTable() throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        flush();
+
+        execute("UPDATE %s SET x = 1, y = 1 WHERE k = 0");
+
+        flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        compact();
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1));
+    }
+
+    /**
+     * Similarly to testAlterDropEmptySSTable, checks we don't return empty rows from queries (testAlterDropEmptySSTable
+     * tests the compaction case).
+     */
+    @Test
+    public void testAlterOnlyColumnBehaviorWithFlush() throws Throwable
+    {
+        testAlterOnlyColumnBehaviorWithFlush(true);
+        testAlterOnlyColumnBehaviorWithFlush(false);
+    }
+
+    private void testAlterOnlyColumnBehaviorWithFlush(boolean flushAfterInsert) throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1, null));
+
+        if (flushAfterInsert)
+            flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        assertEmpty(execute("SELECT * FROM %s"));
+    }
 }


[5/6] cassandra git commit: Merge branch 'cassandra-3.0' into cassandra-3.11

Posted by sl...@apache.org.
Merge branch 'cassandra-3.0' into cassandra-3.11

* cassandra-3.0:
  Dropping column results in "corrupt" SSTable


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

Branch: refs/heads/cassandra-3.11
Commit: c0ac928d985a59d00310fed124c5a2b3655db8e0
Parents: ee7023e 5262bb1
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 27 11:59:06 2017 +0200
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 11:59:06 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  3 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 144 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/CHANGES.txt
----------------------------------------------------------------------
diff --cc CHANGES.txt
index 071dd1a,0b1bb01..98ed992
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@@ -1,20 -1,5 +1,21 @@@
 -3.0.13
 +3.11.0
 + * unittest CipherFactoryTest failed on MacOS (CASSANDRA-13370)
 + * Forbid SELECT restrictions and CREATE INDEX over non-frozen UDT columns (CASSANDRA-13247)
 + * Default logging we ship will incorrectly print "?:?" for "%F:%L" pattern (CASSANDRA-13317)
 + * Possible AssertionError in UnfilteredRowIteratorWithLowerBound (CASSANDRA-13366)
 + * Support unaligned memory access for AArch64 (CASSANDRA-13326)
 + * Improve SASI range iterator efficiency on intersection with an empty range (CASSANDRA-12915).
 + * Fix equality comparisons of columns using the duration type (CASSANDRA-13174)
 + * Obfuscate password in stress-graphs (CASSANDRA-12233)
 + * Move to FastThreadLocalThread and FastThreadLocal (CASSANDRA-13034)
 + * nodetool stopdaemon errors out (CASSANDRA-13030)
 + * Tables in system_distributed should not use gcgs of 0 (CASSANDRA-12954)
 + * Fix primary index calculation for SASI (CASSANDRA-12910)
 + * More fixes to the TokenAllocator (CASSANDRA-12990)
 + * NoReplicationTokenAllocator should work with zero replication factor (CASSANDRA-12983)
 + * Address message coalescing regression (CASSANDRA-12676)
 +Merged from 3.0:
+  * Dropping column results in "corrupt" SSTable (CASSANDRA-13337)
   * Bugs handling range tombstones in the sstable iterators (CASSANDRA-13340)
   * Fix CONTAINS filtering for null collections (CASSANDRA-13246)
   * Applying: Use a unique metric reservoir per test run when using Cassandra-wide metrics residing in MBeans (CASSANDRA-13216)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
----------------------------------------------------------------------


[6/6] cassandra git commit: Merge branch 'cassandra-3.11' into trunk

Posted by sl...@apache.org.
Merge branch 'cassandra-3.11' into trunk

* cassandra-3.11:
  Dropping column results in "corrupt" SSTable


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

Branch: refs/heads/trunk
Commit: 3dabeeaa29bfb6757e79f17031cb467b7b052fcd
Parents: 23cd27f c0ac928
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 27 12:04:20 2017 +0200
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 12:04:20 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  4 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 145 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


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

http://git-wip-us.apache.org/repos/asf/cassandra/blob/3dabeeaa/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/3dabeeaa/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------
diff --cc src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
index e67d164,88d415b..837562a
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
@@@ -223,8 -228,12 +223,10 @@@ public class SSTableReversedIterator ex
                     && !stopReadingDisk())
              {
                  Unfiltered unfiltered = deserializer.readNext();
-                 buffer.add(unfiltered);
+                 // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
 -                if (!unfiltered.isEmpty() && (!isFirst || includeFirst))
++                if (!unfiltered.isEmpty())
+                     buffer.add(unfiltered);
  
 -                isFirst = false;
 -
                  if (unfiltered.isRangeTombstoneMarker())
                      updateOpenMarker((RangeTombstoneMarker)unfiltered);
              }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/3dabeeaa/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------
diff --cc src/java/org/apache/cassandra/db/rows/Unfiltered.java
index 39d9e75,e75c632..e1158e2
--- a/src/java/org/apache/cassandra/db/rows/Unfiltered.java
+++ b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
@@@ -53,11 -53,13 +53,13 @@@ public interface Unfiltered extends Clu
       * invalid (some value is invalid for its column type, or some field
       * is nonsensical).
       */
 -    public void validateData(CFMetaData metadata);
 +    public void validateData(TableMetadata metadata);
  
+     public boolean isEmpty();
+ 
 -    public String toString(CFMetaData metadata);
 -    public String toString(CFMetaData metadata, boolean fullDetails);
 -    public String toString(CFMetaData metadata, boolean includeClusterKeys, boolean fullDetails);
 +    public String toString(TableMetadata metadata);
 +    public String toString(TableMetadata metadata, boolean fullDetails);
 +    public String toString(TableMetadata metadata, boolean includeClusterKeys, boolean fullDetails);
  
      default boolean isRow()
      {

http://git-wip-us.apache.org/repos/asf/cassandra/blob/3dabeeaa/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/3dabeeaa/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
----------------------------------------------------------------------


[2/6] cassandra git commit: Dropping column results in "corrupt" SSTable

Posted by sl...@apache.org.
Dropping column results in "corrupt" SSTable

patch by Sylvain Lebresne; reviewed by Alex Petrov for CASSANDRA-13337


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

Branch: refs/heads/cassandra-3.11
Commit: 5262bb17b46fc8c02f9f836ddf9317d0de2698cd
Parents: 6311622
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 20 15:49:27 2017 +0100
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 11:58:32 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  3 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 144 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index 2c5573a..0b1bb01 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 3.0.13
+ * Dropping column results in "corrupt" SSTable (CASSANDRA-13337)
  * Bugs handling range tombstones in the sstable iterators (CASSANDRA-13340)
  * Fix CONTAINS filtering for null collections (CASSANDRA-13246)
  * Applying: Use a unique metric reservoir per test run when using Cassandra-wide metrics residing in MBeans (CASSANDRA-13216)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
index 9bcca48..fa337c0 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
@@ -123,20 +123,27 @@ public class SSTableIterator extends AbstractSSTableIterator
         {
             assert deserializer != null;
 
-            // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
-            // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
-            // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
-            // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
-            // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
-            // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
-            // an open marker" code in hasNextInterna
-            if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
-                return null;
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
+                // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
+                // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
+                // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
+                // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
+                // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
+                // an open marker" code in hasNextInterna
+                if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
+                    return null;
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
 
         protected boolean hasNextInternal() throws IOException
@@ -256,24 +263,31 @@ public class SSTableIterator extends AbstractSSTableIterator
         @Override
         protected Unfiltered computeNext() throws IOException
         {
-            // Our previous read might have made us cross an index block boundary. If so, update our informations.
-            // If we read from the beginning of the partition, this is also what will initialize the index state.
-            indexState.updateBlock();
-
-            // Return the next unfiltered unless we've reached the end, or we're beyond our slice
-            // end (note that unless we're on the last block for the slice, there is no point
-            // in checking the slice end).
-            if (indexState.isDone()
-                || indexState.currentBlockIdx() > lastBlockIdx
-                || !deserializer.hasNext()
-                || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
-                return null;
-
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // Our previous read might have made us cross an index block boundary. If so, update our informations.
+                // If we read from the beginning of the partition, this is also what will initialize the index state.
+                indexState.updateBlock();
+
+                // Return the next unfiltered unless we've reached the end, or we're beyond our slice
+                // end (note that unless we're on the last block for the slice, there is no point
+                // in checking the slice end).
+                if (indexState.isDone()
+                    || indexState.currentBlockIdx() > lastBlockIdx
+                    || !deserializer.hasNext()
+                    || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
+                    return null;
+
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
index 49dc82a..4bb7fe8 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
@@ -213,7 +213,8 @@ public class SSTableReversedIterator extends AbstractSSTableIterator
                    && !stopReadingDisk())
             {
                 Unfiltered unfiltered = deserializer.readNext();
-                if (!isFirst || includeFirst)
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (!unfiltered.isEmpty() && (!isFirst || includeFirst))
                     buffer.add(unfiltered);
 
                 isFirst = false;

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
index 1cd5fb4..c4c9f7f 100644
--- a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
+++ b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
@@ -49,6 +49,12 @@ public interface RangeTombstoneMarker extends Unfiltered
 
     public RangeTombstoneMarker copy(AbstractAllocator allocator);
 
+    default public boolean isEmpty()
+    {
+        // There is no such thing as an empty marker
+        return false;
+    }
+
     /**
      * Utility class to help merging range tombstone markers coming from multiple inputs (UnfilteredRowIterators).
      * <p>

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/Unfiltered.java b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
index 9d96137..9511eeb 100644
--- a/src/java/org/apache/cassandra/db/rows/Unfiltered.java
+++ b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
@@ -55,6 +55,8 @@ public interface Unfiltered extends Clusterable
      */
     public void validateData(CFMetaData metadata);
 
+    public boolean isEmpty();
+
     public String toString(CFMetaData metadata);
     public String toString(CFMetaData metadata, boolean fullDetails);
     public String toString(CFMetaData metadata, boolean includeClusterKeys, boolean fullDetails);

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
index dc6f187..bdc8388 100644
--- a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
+++ b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
@@ -350,9 +350,44 @@ public class UnfilteredSerializer
         return 1;
     }
 
+    /**
+     * Deserialize an {@link Unfiltered} from the provided input.
+     *
+     * @param in the input from which to deserialize.
+     * @param header serialization header corresponding to the serialized data.
+     * @param helper the helper to use for deserialization.
+     * @param builder a row builder, passed here so we don't allocate a new one for every new row.
+     * @return the deserialized {@link Unfiltered} or {@code null} if we've read the end of a partition. This method is
+     * guaranteed to never return empty rows.
+     */
     public Unfiltered deserialize(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
     throws IOException
     {
+        while (true)
+        {
+            Unfiltered unfiltered = deserializeOne(in, header, helper, builder);
+            if (unfiltered == null)
+                return null;
+
+            // Skip empty rows, see deserializeOne javadoc
+            if (!unfiltered.isEmpty())
+                return unfiltered;
+        }
+    }
+
+    /**
+     * Deserialize a single {@link Unfiltered} from the provided input.
+     * <p>
+     * <b>WARNING:</b> this can return an empty row because it's possible there is a row serialized, but that row only
+     * contains data for dropped columns, see CASSANDRA-13337. But as most code expect rows to not be empty, this isn't
+     * meant to be exposed publicly.
+     *
+     * But as {@link UnfilteredRowIterator} should not return empty
+     * rows, this mean consumer of this method should make sure to skip said empty rows.
+     */
+    private Unfiltered deserializeOne(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
+    throws IOException
+    {
         // It wouldn't be wrong per-se to use an unsorted builder, but it would be inefficient so make sure we don't do it by mistake
         assert builder.isSorted();
 
@@ -374,13 +409,7 @@ public class UnfilteredSerializer
                 throw new IOException("Corrupt flags value for unfiltered partition (isStatic flag set): " + flags);
 
             builder.newRow(Clustering.serializer.deserialize(in, helper.version, header.clusteringTypes()));
-            Row row = deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
-            // we do not write empty rows because Rows.collectStats(), called by BTW.applyToRow(), asserts that rows are not empty
-            // if we don't throw here, then later the very same assertion in Rows.collectStats() will fail compactions
-            // see BlackListingCompactionsTest and CASSANDRA-9530 for details
-            if (row.isEmpty())
-                throw new IOException("Corrupt empty row found in unfiltered partition");
-            return row;
+            return deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
         }
     }
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/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 245be30..c48ffe5 100644
--- a/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
+++ b/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
@@ -355,4 +355,55 @@ public class AlterTest extends CQLTester
             assertEquals(errorMsg, e.getMessage());
         }
     }
+
+    /**
+     * Test for CASSANDRA-13337. Checks that dropping a column when a sstable contains only data for that column
+     * works properly.
+     */
+    @Test
+    public void testAlterDropEmptySSTable() throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        flush();
+
+        execute("UPDATE %s SET x = 1, y = 1 WHERE k = 0");
+
+        flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        compact();
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1));
+    }
+
+    /**
+     * Similarly to testAlterDropEmptySSTable, checks we don't return empty rows from queries (testAlterDropEmptySSTable
+     * tests the compaction case).
+     */
+    @Test
+    public void testAlterOnlyColumnBehaviorWithFlush() throws Throwable
+    {
+        testAlterOnlyColumnBehaviorWithFlush(true);
+        testAlterOnlyColumnBehaviorWithFlush(false);
+    }
+
+    private void testAlterOnlyColumnBehaviorWithFlush(boolean flushAfterInsert) throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1, null));
+
+        if (flushAfterInsert)
+            flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        assertEmpty(execute("SELECT * FROM %s"));
+    }
 }


[3/6] cassandra git commit: Dropping column results in "corrupt" SSTable

Posted by sl...@apache.org.
Dropping column results in "corrupt" SSTable

patch by Sylvain Lebresne; reviewed by Alex Petrov for CASSANDRA-13337


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

Branch: refs/heads/trunk
Commit: 5262bb17b46fc8c02f9f836ddf9317d0de2698cd
Parents: 6311622
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 20 15:49:27 2017 +0100
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 11:58:32 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  3 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 144 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index 2c5573a..0b1bb01 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 3.0.13
+ * Dropping column results in "corrupt" SSTable (CASSANDRA-13337)
  * Bugs handling range tombstones in the sstable iterators (CASSANDRA-13340)
  * Fix CONTAINS filtering for null collections (CASSANDRA-13246)
  * Applying: Use a unique metric reservoir per test run when using Cassandra-wide metrics residing in MBeans (CASSANDRA-13216)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
index 9bcca48..fa337c0 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
@@ -123,20 +123,27 @@ public class SSTableIterator extends AbstractSSTableIterator
         {
             assert deserializer != null;
 
-            // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
-            // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
-            // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
-            // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
-            // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
-            // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
-            // an open marker" code in hasNextInterna
-            if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
-                return null;
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // We use a same reasoning as in handlePreSliceData regarding the strictness of the inequality below.
+                // We want to exclude deserialized unfiltered equal to end, because 1) we won't miss any rows since those
+                // woudn't be equal to a slice bound and 2) a end bound can be equal to a start bound
+                // (EXCL_END(x) == INCL_START(x) for instance) and in that case we don't want to return start bound because
+                // it's fundamentally excluded. And if the bound is a  end (for a range tombstone), it means it's exactly
+                // our slice end, but in that  case we will properly close the range tombstone anyway as part of our "close
+                // an open marker" code in hasNextInterna
+                if (!deserializer.hasNext() || deserializer.compareNextTo(end) >= 0)
+                    return null;
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
 
         protected boolean hasNextInternal() throws IOException
@@ -256,24 +263,31 @@ public class SSTableIterator extends AbstractSSTableIterator
         @Override
         protected Unfiltered computeNext() throws IOException
         {
-            // Our previous read might have made us cross an index block boundary. If so, update our informations.
-            // If we read from the beginning of the partition, this is also what will initialize the index state.
-            indexState.updateBlock();
-
-            // Return the next unfiltered unless we've reached the end, or we're beyond our slice
-            // end (note that unless we're on the last block for the slice, there is no point
-            // in checking the slice end).
-            if (indexState.isDone()
-                || indexState.currentBlockIdx() > lastBlockIdx
-                || !deserializer.hasNext()
-                || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
-                return null;
-
-
-            Unfiltered next = deserializer.readNext();
-            if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
-                updateOpenMarker((RangeTombstoneMarker)next);
-            return next;
+            while (true)
+            {
+                // Our previous read might have made us cross an index block boundary. If so, update our informations.
+                // If we read from the beginning of the partition, this is also what will initialize the index state.
+                indexState.updateBlock();
+
+                // Return the next unfiltered unless we've reached the end, or we're beyond our slice
+                // end (note that unless we're on the last block for the slice, there is no point
+                // in checking the slice end).
+                if (indexState.isDone()
+                    || indexState.currentBlockIdx() > lastBlockIdx
+                    || !deserializer.hasNext()
+                    || (indexState.currentBlockIdx() == lastBlockIdx && deserializer.compareNextTo(end) >= 0))
+                    return null;
+
+
+                Unfiltered next = deserializer.readNext();
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (next.isEmpty())
+                    continue;
+
+                if (next.kind() == Unfiltered.Kind.RANGE_TOMBSTONE_MARKER)
+                    updateOpenMarker((RangeTombstoneMarker) next);
+                return next;
+            }
         }
     }
 }

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
index 49dc82a..4bb7fe8 100644
--- a/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
+++ b/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
@@ -213,7 +213,8 @@ public class SSTableReversedIterator extends AbstractSSTableIterator
                    && !stopReadingDisk())
             {
                 Unfiltered unfiltered = deserializer.readNext();
-                if (!isFirst || includeFirst)
+                // We may get empty row for the same reason expressed on UnfilteredSerializer.deserializeOne.
+                if (!unfiltered.isEmpty() && (!isFirst || includeFirst))
                     buffer.add(unfiltered);
 
                 isFirst = false;

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
index 1cd5fb4..c4c9f7f 100644
--- a/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
+++ b/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
@@ -49,6 +49,12 @@ public interface RangeTombstoneMarker extends Unfiltered
 
     public RangeTombstoneMarker copy(AbstractAllocator allocator);
 
+    default public boolean isEmpty()
+    {
+        // There is no such thing as an empty marker
+        return false;
+    }
+
     /**
      * Utility class to help merging range tombstone markers coming from multiple inputs (UnfilteredRowIterators).
      * <p>

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/Unfiltered.java b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
index 9d96137..9511eeb 100644
--- a/src/java/org/apache/cassandra/db/rows/Unfiltered.java
+++ b/src/java/org/apache/cassandra/db/rows/Unfiltered.java
@@ -55,6 +55,8 @@ public interface Unfiltered extends Clusterable
      */
     public void validateData(CFMetaData metadata);
 
+    public boolean isEmpty();
+
     public String toString(CFMetaData metadata);
     public String toString(CFMetaData metadata, boolean fullDetails);
     public String toString(CFMetaData metadata, boolean includeClusterKeys, boolean fullDetails);

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
index dc6f187..bdc8388 100644
--- a/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
+++ b/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
@@ -350,9 +350,44 @@ public class UnfilteredSerializer
         return 1;
     }
 
+    /**
+     * Deserialize an {@link Unfiltered} from the provided input.
+     *
+     * @param in the input from which to deserialize.
+     * @param header serialization header corresponding to the serialized data.
+     * @param helper the helper to use for deserialization.
+     * @param builder a row builder, passed here so we don't allocate a new one for every new row.
+     * @return the deserialized {@link Unfiltered} or {@code null} if we've read the end of a partition. This method is
+     * guaranteed to never return empty rows.
+     */
     public Unfiltered deserialize(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
     throws IOException
     {
+        while (true)
+        {
+            Unfiltered unfiltered = deserializeOne(in, header, helper, builder);
+            if (unfiltered == null)
+                return null;
+
+            // Skip empty rows, see deserializeOne javadoc
+            if (!unfiltered.isEmpty())
+                return unfiltered;
+        }
+    }
+
+    /**
+     * Deserialize a single {@link Unfiltered} from the provided input.
+     * <p>
+     * <b>WARNING:</b> this can return an empty row because it's possible there is a row serialized, but that row only
+     * contains data for dropped columns, see CASSANDRA-13337. But as most code expect rows to not be empty, this isn't
+     * meant to be exposed publicly.
+     *
+     * But as {@link UnfilteredRowIterator} should not return empty
+     * rows, this mean consumer of this method should make sure to skip said empty rows.
+     */
+    private Unfiltered deserializeOne(DataInputPlus in, SerializationHeader header, SerializationHelper helper, Row.Builder builder)
+    throws IOException
+    {
         // It wouldn't be wrong per-se to use an unsorted builder, but it would be inefficient so make sure we don't do it by mistake
         assert builder.isSorted();
 
@@ -374,13 +409,7 @@ public class UnfilteredSerializer
                 throw new IOException("Corrupt flags value for unfiltered partition (isStatic flag set): " + flags);
 
             builder.newRow(Clustering.serializer.deserialize(in, helper.version, header.clusteringTypes()));
-            Row row = deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
-            // we do not write empty rows because Rows.collectStats(), called by BTW.applyToRow(), asserts that rows are not empty
-            // if we don't throw here, then later the very same assertion in Rows.collectStats() will fail compactions
-            // see BlackListingCompactionsTest and CASSANDRA-9530 for details
-            if (row.isEmpty())
-                throw new IOException("Corrupt empty row found in unfiltered partition");
-            return row;
+            return deserializeRowBody(in, header, helper, flags, extendedFlags, builder);
         }
     }
 

http://git-wip-us.apache.org/repos/asf/cassandra/blob/5262bb17/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 245be30..c48ffe5 100644
--- a/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
+++ b/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
@@ -355,4 +355,55 @@ public class AlterTest extends CQLTester
             assertEquals(errorMsg, e.getMessage());
         }
     }
+
+    /**
+     * Test for CASSANDRA-13337. Checks that dropping a column when a sstable contains only data for that column
+     * works properly.
+     */
+    @Test
+    public void testAlterDropEmptySSTable() throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        flush();
+
+        execute("UPDATE %s SET x = 1, y = 1 WHERE k = 0");
+
+        flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        compact();
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1));
+    }
+
+    /**
+     * Similarly to testAlterDropEmptySSTable, checks we don't return empty rows from queries (testAlterDropEmptySSTable
+     * tests the compaction case).
+     */
+    @Test
+    public void testAlterOnlyColumnBehaviorWithFlush() throws Throwable
+    {
+        testAlterOnlyColumnBehaviorWithFlush(true);
+        testAlterOnlyColumnBehaviorWithFlush(false);
+    }
+
+    private void testAlterOnlyColumnBehaviorWithFlush(boolean flushAfterInsert) throws Throwable
+    {
+        createTable("CREATE TABLE %s(k int PRIMARY KEY, x int, y int)");
+
+        execute("UPDATE %s SET x = 1 WHERE k = 0");
+
+        assertRows(execute("SELECT * FROM %s"), row(0, 1, null));
+
+        if (flushAfterInsert)
+            flush();
+
+        execute("ALTER TABLE %s DROP x");
+
+        assertEmpty(execute("SELECT * FROM %s"));
+    }
 }


[4/6] cassandra git commit: Merge branch 'cassandra-3.0' into cassandra-3.11

Posted by sl...@apache.org.
Merge branch 'cassandra-3.0' into cassandra-3.11

* cassandra-3.0:
  Dropping column results in "corrupt" SSTable


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

Branch: refs/heads/trunk
Commit: c0ac928d985a59d00310fed124c5a2b3655db8e0
Parents: ee7023e 5262bb1
Author: Sylvain Lebresne <sy...@datastax.com>
Authored: Mon Mar 27 11:59:06 2017 +0200
Committer: Sylvain Lebresne <sy...@datastax.com>
Committed: Mon Mar 27 11:59:06 2017 +0200

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../db/columniterator/SSTableIterator.java      | 78 ++++++++++++--------
 .../columniterator/SSTableReversedIterator.java |  3 +-
 .../cassandra/db/rows/RangeTombstoneMarker.java |  6 ++
 .../apache/cassandra/db/rows/Unfiltered.java    |  2 +
 .../cassandra/db/rows/UnfilteredSerializer.java | 43 +++++++++--
 .../cql3/validation/operations/AlterTest.java   | 51 +++++++++++++
 7 files changed, 144 insertions(+), 40 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/CHANGES.txt
----------------------------------------------------------------------
diff --cc CHANGES.txt
index 071dd1a,0b1bb01..98ed992
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@@ -1,20 -1,5 +1,21 @@@
 -3.0.13
 +3.11.0
 + * unittest CipherFactoryTest failed on MacOS (CASSANDRA-13370)
 + * Forbid SELECT restrictions and CREATE INDEX over non-frozen UDT columns (CASSANDRA-13247)
 + * Default logging we ship will incorrectly print "?:?" for "%F:%L" pattern (CASSANDRA-13317)
 + * Possible AssertionError in UnfilteredRowIteratorWithLowerBound (CASSANDRA-13366)
 + * Support unaligned memory access for AArch64 (CASSANDRA-13326)
 + * Improve SASI range iterator efficiency on intersection with an empty range (CASSANDRA-12915).
 + * Fix equality comparisons of columns using the duration type (CASSANDRA-13174)
 + * Obfuscate password in stress-graphs (CASSANDRA-12233)
 + * Move to FastThreadLocalThread and FastThreadLocal (CASSANDRA-13034)
 + * nodetool stopdaemon errors out (CASSANDRA-13030)
 + * Tables in system_distributed should not use gcgs of 0 (CASSANDRA-12954)
 + * Fix primary index calculation for SASI (CASSANDRA-12910)
 + * More fixes to the TokenAllocator (CASSANDRA-12990)
 + * NoReplicationTokenAllocator should work with zero replication factor (CASSANDRA-12983)
 + * Address message coalescing regression (CASSANDRA-12676)
 +Merged from 3.0:
+  * Dropping column results in "corrupt" SSTable (CASSANDRA-13337)
   * Bugs handling range tombstones in the sstable iterators (CASSANDRA-13340)
   * Fix CONTAINS filtering for null collections (CASSANDRA-13246)
   * Applying: Use a unique metric reservoir per test run when using Cassandra-wide metrics residing in MBeans (CASSANDRA-13216)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/columniterator/SSTableIterator.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/columniterator/SSTableReversedIterator.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/RangeTombstoneMarker.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/Unfiltered.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/src/java/org/apache/cassandra/db/rows/UnfilteredSerializer.java
----------------------------------------------------------------------

http://git-wip-us.apache.org/repos/asf/cassandra/blob/c0ac928d/test/unit/org/apache/cassandra/cql3/validation/operations/AlterTest.java
----------------------------------------------------------------------