Author: gdusbabek
Date: Thu Jan 21 22:43:30 2010
New Revision: 901902
URL: http://svn.apache.org/viewvc?rev=901902&view=rev
Log:
ensure that all files for a single sstable are streamed to the same directory.
Patch by Gary Dusbabek, reviewed by Jonathan Ellis. CASSANDRA-716
Modified:
incubator/cassandra/branches/cassandra-0.5/CHANGES.txt
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/io/Streaming.java
incubator/cassandra/branches/cassandra-0.5/test/unit/org/apache/cassandra/dht/BootstrapTest.java
Modified: incubator/cassandra/branches/cassandra-0.5/CHANGES.txt
URL:
http://svn.apache.org/viewvc/incubator/cassandra/branches/cassandra-0.5/CHANGES.txt?rev=901902&r1=901901&r2=901902&view=diff
==============================================================================
--- incubator/cassandra/branches/cassandra-0.5/CHANGES.txt (original)
+++ incubator/cassandra/branches/cassandra-0.5/CHANGES.txt Thu Jan 21 22:43:30
2010
@@ -1,3 +1,7 @@
+0.5.1
+ * ensure all files for an sstable are streamed to the same directory.
+ (CASSANDRA-716)
+
0.5.0 final
* avoid attempting to delete temporary bootstrap files twice (CASSANDRA-681)
* fix bogus NaN in nodeprobe cfstats output (CASSANDRA-646)
Modified:
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
URL:
http://svn.apache.org/viewvc/incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/config/DatabaseDescriptor.java?rev=901902&r1=901901&r2=901902&view=diff
==============================================================================
---
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
(original)
+++
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/config/DatabaseDescriptor.java
Thu Jan 21 22:43:30 2010
@@ -846,9 +846,9 @@
return tableLocations;
}
- public static String getDataFileLocationForTable(String table)
+ public synchronized static String getNextAvailableDataLocation()
{
- String dataFileDirectory = dataFileDirectories_[currentIndex_] +
File.separator + table;
+ String dataFileDirectory = dataFileDirectories_[currentIndex_];
currentIndex_ = (currentIndex_ + 1) % dataFileDirectories_.length;
return dataFileDirectory;
}
Modified:
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
URL:
http://svn.apache.org/viewvc/incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java?rev=901902&r1=901901&r2=901902&view=diff
==============================================================================
---
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
(original)
+++
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
Thu Jan 21 22:43:30 2010
@@ -365,7 +365,7 @@
synchronized String getTempSSTablePath()
{
String fname = getTempSSTableFileName();
- return new
File(DatabaseDescriptor.getDataFileLocationForTable(table_),
fname).getAbsolutePath();
+ return new File(DatabaseDescriptor.getNextAvailableDataLocation() +
File.separator + table_, fname).getAbsolutePath();
}
public String getTempSSTableFileName()
Modified:
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/io/Streaming.java
URL:
http://svn.apache.org/viewvc/incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/io/Streaming.java?rev=901902&r1=901901&r2=901902&view=diff
==============================================================================
---
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/io/Streaming.java
(original)
+++
incubator/cassandra/branches/cassandra-0.5/src/java/org/apache/cassandra/io/Streaming.java
Thu Jan 21 22:43:30 2010
@@ -182,6 +182,9 @@
}
Map<String, String> fileNames = getNewNames(streamContexts);
+ Map<String, String> pathNames = new HashMap<String, String>();
+ for (String ssName : fileNames.keySet())
+ pathNames.put(ssName,
DatabaseDescriptor.getNextAvailableDataLocation());
/*
* For each of stream context's in the incoming message
* generate the new file names and store the new file names
@@ -190,7 +193,7 @@
for (StreamContextManager.StreamContext streamContext :
streamContexts )
{
StreamContextManager.StreamStatus streamStatus = new
StreamContextManager.StreamStatus(streamContext.getTargetFile(),
streamContext.getExpectedBytes() );
- String file =
getNewFileNameFromOldContextAndNames(fileNames, streamContext);
+ String file =
getNewFileNameFromOldContextAndNames(fileNames, pathNames, streamContext);
if (logger.isDebugEnabled())
logger.debug("Received Data from : " +
message.getFrom() + " " + streamContext.getTargetFile() + " " + file);
@@ -211,6 +214,7 @@
}
public String getNewFileNameFromOldContextAndNames(Map<String, String>
fileNames,
+ Map<String, String>
pathNames,
StreamContextManager.StreamContext streamContext)
{
File sourceFile = new File( streamContext.getTargetFile() );
@@ -219,12 +223,14 @@
String ssTableNum = piece[1];
String typeOfFile = piece[2];
- String newFileNameExpanded = fileNames.get(
streamContext.getTable() + "-" + cfName + "-" + ssTableNum );
+ String newFileNameExpanded =
fileNames.get(streamContext.getTable() + "-" + cfName + "-" + ssTableNum);
+ String path = pathNames.get(streamContext.getTable() + "-" +
cfName + "-" + ssTableNum);
//Drop type (Data.db) from new FileName
String newFileName = newFileNameExpanded.replace("Data.db",
typeOfFile);
- return
DatabaseDescriptor.getDataFileLocationForTable(streamContext.getTable()) +
File.separator + newFileName;
+ return path + File.separator + streamContext.getTable() +
File.separator + newFileName;
}
+ // todo: this method needs to be private, or package at the very least
for easy unit testing.
public Map<String, String>
getNewNames(StreamContextManager.StreamContext[] streamContexts) throws
IOException
{
/*
Modified:
incubator/cassandra/branches/cassandra-0.5/test/unit/org/apache/cassandra/dht/BootstrapTest.java
URL:
http://svn.apache.org/viewvc/incubator/cassandra/branches/cassandra-0.5/test/unit/org/apache/cassandra/dht/BootstrapTest.java?rev=901902&r1=901901&r2=901902&view=diff
==============================================================================
---
incubator/cassandra/branches/cassandra-0.5/test/unit/org/apache/cassandra/dht/BootstrapTest.java
(original)
+++
incubator/cassandra/branches/cassandra-0.5/test/unit/org/apache/cassandra/dht/BootstrapTest.java
Thu Jan 21 22:43:30 2010
@@ -23,8 +23,10 @@
import java.io.File;
import java.io.IOException;
+import java.util.HashMap;
import java.util.Map;
+import org.apache.cassandra.config.DatabaseDescriptor;
import org.apache.cassandra.net.io.StreamContextManager;
import org.apache.cassandra.io.Streaming;
@@ -36,18 +38,22 @@
public void testGetNewNames() throws IOException
{
StreamContextManager.StreamContext[] streamContexts = new
StreamContextManager.StreamContext[3];
- streamContexts[0] = new
StreamContextManager.StreamContext("/foo/Standard1-500-Data.db", 100,
"Keyspace1");
- streamContexts[1] = new
StreamContextManager.StreamContext("/foo/Standard1-500-Index.db", 100,
"Keyspace1");
- streamContexts[2] = new
StreamContextManager.StreamContext("/foo/Standard1-500-Filter.db", 100,
"Keyspace1");
+ streamContexts[0] = new
StreamContextManager.StreamContext("/baz/foo/Standard1-500-Data.db", 100,
"Keyspace1");
+ streamContexts[1] = new
StreamContextManager.StreamContext("/bar/foo/Standard1-500-Index.db", 100,
"Keyspace1");
+ streamContexts[2] = new
StreamContextManager.StreamContext("/bad/foo/Standard1-500-Filter.db", 100,
"Keyspace1");
Streaming.StreamInitiateVerbHandler bivh = new
Streaming.StreamInitiateVerbHandler();
Map<String, String> fileNames = bivh.getNewNames(streamContexts);
+ Map<String, String> paths = new HashMap<String, String>();
+ for (String ssName : fileNames.keySet())
+ paths.put(ssName,
DatabaseDescriptor.getNextAvailableDataLocation());
+ assertEquals(1, paths.size());
String result = fileNames.get("Keyspace1-Standard1-500");
assertEquals(true, result.contains("Standard1"));
assertEquals(true, result.contains("Data.db"));
assertEquals(1, fileNames.entrySet().size());
- assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames,
streamContexts[0])).getName().matches("Standard1-tmp-\\d+-Data.db"));
- assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames,
streamContexts[1])).getName().matches("Standard1-tmp-\\d+-Index.db"));
- assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames,
streamContexts[2])).getName().matches("Standard1-tmp-\\d+-Filter.db"));
+ assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames, paths,
streamContexts[0])).getName().matches("Standard1-tmp-\\d+-Data.db"));
+ assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames, paths,
streamContexts[1])).getName().matches("Standard1-tmp-\\d+-Index.db"));
+ assertTrue(new
File(bivh.getNewFileNameFromOldContextAndNames(fileNames, paths,
streamContexts[2])).getName().matches("Standard1-tmp-\\d+-Filter.db"));
}
}