Merge branch 'cassandra-2.2' into cassandra-3.0
Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/b623e82c Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/b623e82c Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/b623e82c Branch: refs/heads/cassandra-3.0 Commit: b623e82c4527aeb15f12ae975adb5d3cf864b34d Parents: a479fb0 a48b836 Author: Marcus Eriksson <marc...@apache.org> Authored: Wed Mar 16 09:22:44 2016 +0100 Committer: Marcus Eriksson <marc...@apache.org> Committed: Wed Mar 16 09:23:04 2016 +0100 ---------------------------------------------------------------------- CHANGES.txt | 1 + .../writers/MaxSSTableSizeWriter.java | 27 ++++++++++++-------- 2 files changed, 18 insertions(+), 10 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/cassandra/blob/b623e82c/CHANGES.txt ---------------------------------------------------------------------- diff --cc CHANGES.txt index 3b5e51f,a69164e..13c682f --- a/CHANGES.txt +++ b/CHANGES.txt @@@ -1,12 -1,8 +1,13 @@@ -2.2.6 +3.0.5 + * Fix sstabledump to include missing info in debug output (CASSANDRA-11321) + * Establish and implement canonical bulk reading workload(s) (CASSANDRA-10331) + * Fix paging for IN queries on tables without clustering columns (CASSANDRA-11208) + * Remove recursive call from CompositesSearcher (CASSANDRA-11304) + * Fix filtering on non-primary key columns for queries without index (CASSANDRA-6377) + * Fix sstableloader fail when using materialized view (CASSANDRA-11275) +Merged from 2.2: + * Fix bloom filter sizing with LCS (CASSANDRA-11344) * (cqlsh) Fix error when result is 0 rows with EXPAND ON (CASSANDRA-11092) - * Fix intra-node serialization issue for multicolumn-restrictions (CASSANDRA-11196) - * Non-obsoleting compaction operations over compressed files can impose rate limit on normal reads (CASSANDRA-11301) * Add missing newline at end of bin/cqlsh (CASSANDRA-11325) * Fix AE in nodetool cfstats (backport CASSANDRA-10859) (CASSANDRA-11297) * Unresolved hostname leads to replace being ignored (CASSANDRA-11210) http://git-wip-us.apache.org/repos/asf/cassandra/blob/b623e82c/src/java/org/apache/cassandra/db/compaction/writers/MaxSSTableSizeWriter.java ---------------------------------------------------------------------- diff --cc src/java/org/apache/cassandra/db/compaction/writers/MaxSSTableSizeWriter.java index 142fe87,2bae3b8..b206498 --- a/src/java/org/apache/cassandra/db/compaction/writers/MaxSSTableSizeWriter.java +++ b/src/java/org/apache/cassandra/db/compaction/writers/MaxSSTableSizeWriter.java @@@ -20,10 -20,10 +20,11 @@@ package org.apache.cassandra.db.compact import java.util.Set; import org.apache.cassandra.db.ColumnFamilyStore; +import org.apache.cassandra.db.Directories; import org.apache.cassandra.db.RowIndexEntry; -import org.apache.cassandra.db.compaction.AbstractCompactedRow; +import org.apache.cassandra.db.SerializationHeader; + import org.apache.cassandra.db.compaction.OperationType; +import org.apache.cassandra.db.rows.UnfilteredRowIterator; import org.apache.cassandra.db.lifecycle.LifecycleTransaction; import org.apache.cassandra.io.sstable.Descriptor; import org.apache.cassandra.io.sstable.format.SSTableReader; @@@ -63,39 -45,50 +63,46 @@@ public class MaxSSTableSizeWriter exten this.allSSTables = txn.originals(); this.level = level; this.maxSSTableSize = maxSSTableSize; - long totalSize = cfs.getExpectedCompactedFileSize(nonExpiredSSTables, txn.opType()); - long totalSize = getTotalWriteSize(nonExpiredSSTables, estimatedTotalKeys, cfs, compactionType); ++ ++ long totalSize = getTotalWriteSize(nonExpiredSSTables, estimatedTotalKeys, cfs, txn.opType()); expectedWriteSize = Math.min(maxSSTableSize, totalSize); - estimatedTotalKeys = SSTableReader.getApproximateKeyCount(nonExpiredSSTables); - estimatedSSTables = Math.max(1, estimatedTotalKeys / maxSSTableSize); + estimatedSSTables = Math.max(1, totalSize / maxSSTableSize); - File sstableDirectory = cfs.directories.getLocationForDisk(getWriteDirectory(expectedWriteSize)); - @SuppressWarnings("resource") - SSTableWriter writer = SSTableWriter.create(Descriptor.fromFilename(cfs.getTempSSTablePath(sstableDirectory)), - estimatedTotalKeys / estimatedSSTables, - minRepairedAt, - cfs.metadata, - cfs.partitioner, - new MetadataCollector(allSSTables, cfs.metadata.comparator, level)); - sstableWriter.switchWriter(writer); + } + + /** + * Gets the estimated total amount of data to write during compaction + */ + private static long getTotalWriteSize(Iterable<SSTableReader> nonExpiredSSTables, long estimatedTotalKeys, ColumnFamilyStore cfs, OperationType compactionType) + { + long estimatedKeysBeforeCompaction = 0; + for (SSTableReader sstable : nonExpiredSSTables) + estimatedKeysBeforeCompaction += sstable.estimatedKeys(); + estimatedKeysBeforeCompaction = Math.max(1, estimatedKeysBeforeCompaction); + double estimatedCompactionRatio = (double) estimatedTotalKeys / estimatedKeysBeforeCompaction; + return Math.round(estimatedCompactionRatio * cfs.getExpectedCompactedFileSize(nonExpiredSSTables, compactionType)); } @Override - public boolean append(AbstractCompactedRow row) + public boolean realAppend(UnfilteredRowIterator partition) { - RowIndexEntry rie = sstableWriter.append(row); + RowIndexEntry rie = sstableWriter.append(partition); if (sstableWriter.currentWriter().getOnDiskFilePointer() > maxSSTableSize) - { - File sstableDirectory = cfs.directories.getLocationForDisk(getWriteDirectory(expectedWriteSize)); - @SuppressWarnings("resource") - SSTableWriter writer = SSTableWriter.create(Descriptor.fromFilename(cfs.getTempSSTablePath(sstableDirectory)), - estimatedTotalKeys / estimatedSSTables, - minRepairedAt, - cfs.metadata, - cfs.partitioner, - new MetadataCollector(allSSTables, cfs.metadata.comparator, level)); - - sstableWriter.switchWriter(writer); - } + switchCompactionLocation(getWriteDirectory(expectedWriteSize)); return rie != null; } + + public void switchCompactionLocation(Directories.DataDirectory location) + { + @SuppressWarnings("resource") + SSTableWriter writer = SSTableWriter.create(Descriptor.fromFilename(cfs.getSSTablePath(getDirectories().getLocationForDisk(location))), + estimatedTotalKeys / estimatedSSTables, + minRepairedAt, + cfs.metadata, + new MetadataCollector(allSSTables, cfs.metadata.comparator, level), + SerializationHeader.make(cfs.metadata, nonExpiredSSTables), + txn); + + sstableWriter.switchWriter(writer); + + } - - @Override - public long estimatedKeys() - { - return estimatedTotalKeys; - } }