Author: jbellis Date: Tue Sep 14 15:08:11 2010 New Revision: 996939 URL: http://svn.apache.org/viewvc?rev=996939&view=rev Log: clean up StreamHeader constructor, r/m unused parameter from transferSSTables. patch by jbellis
Modified: cassandra/trunk/src/java/org/apache/cassandra/service/AntiEntropyService.java cassandra/trunk/src/java/org/apache/cassandra/streaming/IncomingStreamReader.java cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamHeader.java cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamOut.java cassandra/trunk/test/unit/org/apache/cassandra/streaming/StreamingTransferTest.java Modified: cassandra/trunk/src/java/org/apache/cassandra/service/AntiEntropyService.java URL: http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/service/AntiEntropyService.java?rev=996939&r1=996938&r2=996939&view=diff ============================================================================== --- cassandra/trunk/src/java/org/apache/cassandra/service/AntiEntropyService.java (original) +++ cassandra/trunk/src/java/org/apache/cassandra/service/AntiEntropyService.java Tue Sep 14 15:08:11 2010 @@ -539,7 +539,7 @@ public class AntiEntropyService protected void runMayThrow() throws Exception { StreamContext context = new StreamContext(request.endpoint); - StreamOut.transferSSTables(context, request.cf.left, sstables, ranges); + StreamOut.transferSSTables(context, sstables, ranges); StreamOutManager.remove(context); } }); Modified: cassandra/trunk/src/java/org/apache/cassandra/streaming/IncomingStreamReader.java URL: http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/streaming/IncomingStreamReader.java?rev=996939&r1=996938&r2=996939&view=diff ============================================================================== --- cassandra/trunk/src/java/org/apache/cassandra/streaming/IncomingStreamReader.java (original) +++ cassandra/trunk/src/java/org/apache/cassandra/streaming/IncomingStreamReader.java Tue Sep 14 15:08:11 2010 @@ -37,6 +37,7 @@ public class IncomingStreamReader private FileStatus streamStatus; private SocketChannel socketChannel; private StreamContext context; + // indicates an transfer initiated by the source, as opposed to one requested by the recipient private boolean initiatedTransfer; public IncomingStreamReader(StreamHeader header, SocketChannel socketChannel) throws IOException Modified: cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamHeader.java URL: http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamHeader.java?rev=996939&r1=996938&r2=996939&view=diff ============================================================================== --- cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamHeader.java (original) +++ cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamHeader.java Tue Sep 14 15:08:11 2010 @@ -46,7 +46,7 @@ public class StreamHeader private PendingFile file; private long sessionId; - // indicates an initiated transfer as opposed to a request + // indicates an transfer initiated by the source, as opposed to one requested by the recipient protected final boolean initiatedTransfer; // this list will only be non-null when the first of a batch of files are being sent. it avoids having to have @@ -55,10 +55,7 @@ public class StreamHeader public StreamHeader(long sessionId, PendingFile file, boolean initiatedTransfer) { - this.sessionId = sessionId; - this.file = file; - this.initiatedTransfer = initiatedTransfer; - pending = null; + this(sessionId, file, null, initiatedTransfer); } public StreamHeader(long sessionId, PendingFile file, List<PendingFile> pending, boolean initiatedTransfer) Modified: cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamOut.java URL: http://svn.apache.org/viewvc/cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamOut.java?rev=996939&r1=996938&r2=996939&view=diff ============================================================================== --- cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamOut.java (original) +++ cassandra/trunk/src/java/org/apache/cassandra/streaming/StreamOut.java Tue Sep 14 15:08:11 2010 @@ -76,7 +76,7 @@ public class StreamOut { Table table = flushSSTable(tableName); // send the matching portion of every sstable in the keyspace - transferSSTables(context, tableName, table.getAllSSTables(), ranges); + transferSSTables(context, table.getAllSSTables(), ranges); } catch (IOException e) { @@ -144,7 +144,7 @@ public class StreamOut /** * Transfers matching portions of a group of sstables from a single table to the target endpoint. */ - public static void transferSSTables(StreamContext context, String table, Collection<SSTableReader> sstables, Collection<Range> ranges) throws IOException + public static void transferSSTables(StreamContext context, Collection<SSTableReader> sstables, Collection<Range> ranges) throws IOException { List<PendingFile> pending = createPendingFiles(sstables, ranges); Modified: cassandra/trunk/test/unit/org/apache/cassandra/streaming/StreamingTransferTest.java URL: http://svn.apache.org/viewvc/cassandra/trunk/test/unit/org/apache/cassandra/streaming/StreamingTransferTest.java?rev=996939&r1=996938&r2=996939&view=diff ============================================================================== --- cassandra/trunk/test/unit/org/apache/cassandra/streaming/StreamingTransferTest.java (original) +++ cassandra/trunk/test/unit/org/apache/cassandra/streaming/StreamingTransferTest.java Tue Sep 14 15:08:11 2010 @@ -34,8 +34,6 @@ import org.apache.cassandra.dht.Range; import org.apache.cassandra.io.sstable.SSTableUtils; import org.apache.cassandra.io.sstable.SSTableReader; import org.apache.cassandra.service.StorageService; -import org.apache.cassandra.streaming.StreamContext; -import org.apache.cassandra.streaming.StreamOut; import org.apache.cassandra.utils.FBUtilities; import org.junit.BeforeClass; @@ -68,7 +66,7 @@ public class StreamingTransferTest exten List<Range> ranges = new ArrayList<Range>(); ranges.add(new Range(p.getMinimumToken(), p.getToken("key".getBytes()))); ranges.add(new Range(p.getToken("key2".getBytes()), p.getMinimumToken())); - StreamOut.transferSSTables(new StreamContext(LOCAL), tablename, Arrays.asList(sstable), ranges); + StreamOut.transferSSTables(new StreamContext(LOCAL), Arrays.asList(sstable), ranges); // confirm that the SSTable was transferred and registered ColumnFamilyStore cfstore = Table.open(tablename).getColumnFamilyStore(cfname); @@ -108,7 +106,7 @@ public class StreamingTransferTest exten List<Range> ranges = new ArrayList<Range>(); ranges.add(new Range(p.getMinimumToken(), p.getToken("transfer1".getBytes()))); ranges.add(new Range(p.getToken("test2".getBytes()), p.getMinimumToken())); - StreamOut.transferSSTables(new StreamContext(LOCAL), tablename, Arrays.asList(sstable, sstable2), ranges); + StreamOut.transferSSTables(new StreamContext(LOCAL), Arrays.asList(sstable, sstable2), ranges); // confirm that the SSTable was transferred and registered ColumnFamilyStore cfstore = Table.open(tablename).getColumnFamilyStore(cfname);