Author: jbellis
Date: Fri Mar 11 02:57:15 2011
New Revision: 1080433
URL: http://svn.apache.org/viewvc?rev=1080433&view=rev
Log:
reduce memory use during streaming of multiple sstables
patch by jbellis; reviewed by mdennis and tested by Joaquin Casares for
CASSANDRA-2301
Modified:
cassandra/branches/cassandra-0.7/CHANGES.txt
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/db/CompactionManager.java
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/io/sstable/SSTableWriter.java
Modified: cassandra/branches/cassandra-0.7/CHANGES.txt
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.7/CHANGES.txt?rev=1080433&r1=1080432&r2=1080433&view=diff
==============================================================================
--- cassandra/branches/cassandra-0.7/CHANGES.txt (original)
+++ cassandra/branches/cassandra-0.7/CHANGES.txt Fri Mar 11 02:57:15 2011
@@ -19,6 +19,7 @@
* fix commitlog replay when flush position refers to data that didn't
get synced before server died (CASSANDRA-2285)
* fix fd leak in sstable2json with non-mmap'd i/o (CASSANDRA-2304)
+ * reduce memory use during streaming of multiple sstables (CASSANDRA-2301)
0.7.3
Modified:
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/db/CompactionManager.java
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/db/CompactionManager.java?rev=1080433&r1=1080432&r2=1080433&view=diff
==============================================================================
---
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/db/CompactionManager.java
(original)
+++
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/db/CompactionManager.java
Fri Mar 11 02:57:15 2011
@@ -927,7 +927,7 @@ public class CompactionManager implement
return executor.submit(runnable);
}
- public Future<SSTableReader> submitSSTableBuild(Descriptor desc)
+ public Future<SSTableReader> submitSSTableBuild(final Descriptor desc)
{
// invalid descriptions due to missing or dropped CFS are handled by
SSTW and StreamInSession.
final SSTableWriter.Builder builder =
SSTableWriter.createBuilder(desc);
@@ -938,7 +938,7 @@ public class CompactionManager implement
compactionLock.lock();
try
{
- executor.beginCompaction(builder.cfs.columnFamily,
builder);
+ executor.beginCompaction(desc.cfname, builder);
return builder.build();
}
finally
Modified:
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/io/sstable/SSTableWriter.java
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/io/sstable/SSTableWriter.java?rev=1080433&r1=1080432&r2=1080433&view=diff
==============================================================================
---
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/io/sstable/SSTableWriter.java
(original)
+++
cassandra/branches/cassandra-0.7/src/java/org/apache/cassandra/io/sstable/SSTableWriter.java
Fri Mar 11 02:57:15 2011
@@ -244,7 +244,7 @@ public class SSTableWriter extends SSTab
public static class Builder implements ICompactionInfo
{
private final Descriptor desc;
- public final ColumnFamilyStore cfs;
+ private final ColumnFamilyStore cfs;
private BufferedRandomAccessFile dfile;
public Builder(Descriptor desc)
@@ -252,6 +252,14 @@ public class SSTableWriter extends SSTab
this.desc = desc;
cfs = Table.open(desc.ksname).getColumnFamilyStore(desc.cfname);
+ }
+
+ // lazy-initialize the file to avoid opening it until it's actually
executing on the CompactionManager,
+ // since the 8MB buffers can use up heap quickly
+ private void maybeOpenFile()
+ {
+ if (dfile != null)
+ return;
try
{
dfile = new BufferedRandomAccessFile(new
File(desc.filenameFor(SSTable.COMPONENT_DATA)), "r", 8 * 1024 * 1024, true);
@@ -266,6 +274,8 @@ public class SSTableWriter extends SSTab
{
if (cfs.isInvalid())
return null;
+ maybeOpenFile();
+
File ifile = new File(desc.filenameFor(SSTable.COMPONENT_INDEX));
File ffile = new File(desc.filenameFor(SSTable.COMPONENT_FILTER));
assert !ifile.exists();
@@ -332,8 +342,10 @@ public class SSTableWriter extends SSTab
public long getTotalBytes()
{
+ maybeOpenFile();
try
{
+ // (length is still valid post-close)
return dfile.length();
}
catch (IOException e)
@@ -344,6 +356,8 @@ public class SSTableWriter extends SSTab
public long getBytesComplete()
{
+ maybeOpenFile();
+ // (getFilePointer is still valid post-close)
return dfile.getFilePointer();
}