Repository: cassandra Updated Branches: refs/heads/cassandra-2.1 0276c2aab -> 739f3e37c
Log the partition key with tombstone warnings patch by Lyben Todorov; reviewed by Aleksey Yeschenko for CASSANDRA-8561 Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/739f3e37 Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/739f3e37 Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/739f3e37 Branch: refs/heads/cassandra-2.1 Commit: 739f3e37cc988626038a0c98642975071a8ece78 Parents: 0276c2a Author: Lyuben Todorov <[email protected]> Authored: Sun May 3 23:14:41 2015 +0300 Committer: Aleksey Yeschenko <[email protected]> Committed: Sun May 3 23:14:41 2015 +0300 ---------------------------------------------------------------------- CHANGES.txt | 1 + .../org/apache/cassandra/db/RowIteratorFactory.java | 2 +- .../apache/cassandra/db/filter/ExtendedFilter.java | 2 +- .../apache/cassandra/db/filter/IDiskAtomFilter.java | 2 +- .../cassandra/db/filter/NamesQueryFilter.java | 2 +- .../org/apache/cassandra/db/filter/QueryFilter.java | 13 +++++++------ .../cassandra/db/filter/SliceQueryFilter.java | 16 ++++++++++++---- 7 files changed, 24 insertions(+), 14 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/CHANGES.txt ---------------------------------------------------------------------- diff --git a/CHANGES.txt b/CHANGES.txt index a8cd1db..0593e2b 100644 --- a/CHANGES.txt +++ b/CHANGES.txt @@ -1,4 +1,5 @@ 2.1.6 + * Log the partition key with tombstone warnings (CASSANDRA-8561) * Reduce runWithCompactionsDisabled poll interval to 1ms (CASSANDRA-9271) * Fix PITR commitlog replay (CASSANDRA-9195) * GCInspector logs very different times (CASSANDRA-9124) http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/RowIteratorFactory.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/RowIteratorFactory.java b/src/java/org/apache/cassandra/db/RowIteratorFactory.java index 5bd2d9b..d6c8905 100644 --- a/src/java/org/apache/cassandra/db/RowIteratorFactory.java +++ b/src/java/org/apache/cassandra/db/RowIteratorFactory.java @@ -96,7 +96,7 @@ public class RowIteratorFactory if (cached == null || !cfs.isFilterFullyCoveredBy(filter, cached, now)) { // not cached: collate - QueryFilter.collateOnDiskAtom(returnCF, colIters, filter, gcBefore, now); + QueryFilter.collateOnDiskAtom(returnCF, colIters, filter, key, gcBefore, now); } else { http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/filter/ExtendedFilter.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/filter/ExtendedFilter.java b/src/java/org/apache/cassandra/db/filter/ExtendedFilter.java index f566c1d..e46dca7 100644 --- a/src/java/org/apache/cassandra/db/filter/ExtendedFilter.java +++ b/src/java/org/apache/cassandra/db/filter/ExtendedFilter.java @@ -304,7 +304,7 @@ public abstract class ExtendedFilter ColumnFamily pruned = data.cloneMeShallow(); IDiskAtomFilter filter = dataRange.columnFilter(rowKey.getKey()); Iterator<Cell> iter = filter.getColumnIterator(data); - filter.collectReducedColumns(pruned, QueryFilter.gatherTombstones(pruned, iter), cfs.gcBefore(timestamp), timestamp); + filter.collectReducedColumns(pruned, QueryFilter.gatherTombstones(pruned, iter), rowKey, cfs.gcBefore(timestamp), timestamp); return pruned; } http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/filter/IDiskAtomFilter.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/filter/IDiskAtomFilter.java b/src/java/org/apache/cassandra/db/filter/IDiskAtomFilter.java index 3750c75..da5663b 100644 --- a/src/java/org/apache/cassandra/db/filter/IDiskAtomFilter.java +++ b/src/java/org/apache/cassandra/db/filter/IDiskAtomFilter.java @@ -69,7 +69,7 @@ public interface IDiskAtomFilter * by the filter code, which should have some limit on the number of columns * to avoid running out of memory on large rows. */ - public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, int gcBefore, long now); + public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, DecoratedKey key, int gcBefore, long now); public Comparator<Cell> getColumnComparator(CellNameType comparator); http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/filter/NamesQueryFilter.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/filter/NamesQueryFilter.java b/src/java/org/apache/cassandra/db/filter/NamesQueryFilter.java index 77a5dc7..f31d1a0 100644 --- a/src/java/org/apache/cassandra/db/filter/NamesQueryFilter.java +++ b/src/java/org/apache/cassandra/db/filter/NamesQueryFilter.java @@ -94,7 +94,7 @@ public class NamesQueryFilter implements IDiskAtomFilter return new SSTableNamesIterator(sstable, file, key, columns, indexEntry); } - public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, int gcBefore, long now) + public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, DecoratedKey key, int gcBefore, long now) { DeletionInfo.InOrderTester tester = container.inOrderDeletionTester(); while (reducedColumns.hasNext()) http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/filter/QueryFilter.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/filter/QueryFilter.java b/src/java/org/apache/cassandra/db/filter/QueryFilter.java index f58fa9f..db531a5 100644 --- a/src/java/org/apache/cassandra/db/filter/QueryFilter.java +++ b/src/java/org/apache/cassandra/db/filter/QueryFilter.java @@ -66,35 +66,37 @@ public class QueryFilter List<? extends Iterator<? extends OnDiskAtom>> toCollate, int gcBefore) { - collateOnDiskAtom(returnCF, toCollate, filter, gcBefore, timestamp); + collateOnDiskAtom(returnCF, toCollate, filter, this.key, gcBefore, timestamp); } public static void collateOnDiskAtom(ColumnFamily returnCF, List<? extends Iterator<? extends OnDiskAtom>> toCollate, IDiskAtomFilter filter, + DecoratedKey key, int gcBefore, long timestamp) { List<Iterator<Cell>> filteredIterators = new ArrayList<>(toCollate.size()); for (Iterator<? extends OnDiskAtom> iter : toCollate) filteredIterators.add(gatherTombstones(returnCF, iter)); - collateColumns(returnCF, filteredIterators, filter, gcBefore, timestamp); + collateColumns(returnCF, filteredIterators, filter, key, gcBefore, timestamp); } // When there is only a single source of atoms, we can skip the collate step public void collateOnDiskAtom(ColumnFamily returnCF, Iterator<? extends OnDiskAtom> toCollate, int gcBefore) { - filter.collectReducedColumns(returnCF, gatherTombstones(returnCF, toCollate), gcBefore, timestamp); + filter.collectReducedColumns(returnCF, gatherTombstones(returnCF, toCollate), this.key, gcBefore, timestamp); } public void collateColumns(ColumnFamily returnCF, List<? extends Iterator<Cell>> toCollate, int gcBefore) { - collateColumns(returnCF, toCollate, filter, gcBefore, timestamp); + collateColumns(returnCF, toCollate, filter, this.key, gcBefore, timestamp); } public static void collateColumns(ColumnFamily returnCF, List<? extends Iterator<Cell>> toCollate, IDiskAtomFilter filter, + DecoratedKey key, int gcBefore, long timestamp) { @@ -103,8 +105,7 @@ public class QueryFilter Iterator<Cell> reduced = toCollate.size() == 1 ? toCollate.get(0) : MergeIterator.get(toCollate, comparator, getReducer(comparator)); - - filter.collectReducedColumns(returnCF, reduced, gcBefore, timestamp); + filter.collectReducedColumns(returnCF, reduced, key, gcBefore, timestamp); } private static MergeIterator.Reducer<Cell, Cell> getReducer(final Comparator<Cell> comparator) http://git-wip-us.apache.org/repos/asf/cassandra/blob/739f3e37/src/java/org/apache/cassandra/db/filter/SliceQueryFilter.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/filter/SliceQueryFilter.java b/src/java/org/apache/cassandra/db/filter/SliceQueryFilter.java index 540aeef..38947bf 100644 --- a/src/java/org/apache/cassandra/db/filter/SliceQueryFilter.java +++ b/src/java/org/apache/cassandra/db/filter/SliceQueryFilter.java @@ -195,7 +195,7 @@ public class SliceQueryFilter implements IDiskAtomFilter return reversed ? comparator.columnReverseComparator() : comparator.columnComparator(false); } - public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, int gcBefore, long now) + public void collectReducedColumns(ColumnFamily container, Iterator<Cell> reducedColumns, DecoratedKey key, int gcBefore, long now) { columnCounter = columnCounter(container.getComparator(), now); DeletionInfo.InOrderTester tester = container.deletionInfo().inOrderTester(reversed); @@ -224,10 +224,11 @@ public class SliceQueryFilter implements IDiskAtomFilter } Tracing.trace("Read {} live and {} tombstoned cells", columnCounter.live(), columnCounter.ignored()); - if (respectTombstoneThresholds() && columnCounter.ignored() > DatabaseDescriptor.getTombstoneWarnThreshold()) + if (logger.isWarnEnabled() && respectTombstoneThresholds() && columnCounter.ignored() > DatabaseDescriptor.getTombstoneWarnThreshold()) { StringBuilder sb = new StringBuilder(); CellNameType type = container.metadata().comparator; + for (ColumnSlice sl : slices) { assert sl != null; @@ -239,8 +240,15 @@ public class SliceQueryFilter implements IDiskAtomFilter sb.append(']'); } - logger.warn("Read {} live and {} tombstoned cells in {}.{} (see tombstone_warn_threshold). {} columns was requested, slices={}", - columnCounter.live(), columnCounter.ignored(), container.metadata().ksName, container.metadata().cfName, count, sb); + String msg = String.format("Read %d live and %d tombstoned cells in %s.%s for key: %1.512s (see tombstone_warn_threshold). %d columns were requested, slices=%1.512s", + columnCounter.live(), + columnCounter.ignored(), + container.metadata().ksName, + container.metadata().cfName, + container.metadata().getKeyValidator().getString(key.getKey()), + count, + sb); + logger.warn(msg); } }
