You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by jb...@apache.org on 2009/04/24 02:46:19 UTC

svn commit: r768108 - in /incubator/cassandra/trunk/src/org/apache/cassandra: db/Row.java db/SuperColumn.java service/ReadRepairManager.java service/ReadResponseResolver.java service/StorageProxy.java

Author: jbellis
Date: Fri Apr 24 00:46:19 2009
New Revision: 768108

URL: http://svn.apache.org/viewvc?rev=768108&view=rev
Log:
clean up Read Repair code to use non-deprecated APIs.
patch by jbellis; reviewed by Jun Rao for #87.

Modified:
    incubator/cassandra/trunk/src/org/apache/cassandra/db/Row.java
    incubator/cassandra/trunk/src/org/apache/cassandra/db/SuperColumn.java
    incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadRepairManager.java
    incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadResponseResolver.java
    incubator/cassandra/trunk/src/org/apache/cassandra/service/StorageProxy.java

Modified: incubator/cassandra/trunk/src/org/apache/cassandra/db/Row.java
URL: http://svn.apache.org/viewvc/incubator/cassandra/trunk/src/org/apache/cassandra/db/Row.java?rev=768108&r1=768107&r2=768108&view=diff
==============================================================================
--- incubator/cassandra/trunk/src/org/apache/cassandra/db/Row.java (original)
+++ incubator/cassandra/trunk/src/org/apache/cassandra/db/Row.java Fri Apr 24 00:46:19 2009
@@ -133,32 +133,30 @@
      * and return the resultant row. This assumes that the row that
      * is being submitted is a super set of the current row so
      * it only calculates additional
-     * difference and does not take care of what needs to be delted from the current row to make
+     * difference and does not take care of what needs to be removed from the current row to make
      * it same as the input row.
      */
-    public Row diff(Row row)
+    public Row diff(Row rowNew)
     {
         Row rowDiff = new Row(key_);
-        Map<String, ColumnFamily> columnFamilies = row.getColumnFamilyMap();
-        Set<String> cfNames = columnFamilies.keySet();
 
-        for (String cfName : cfNames)
+        for (ColumnFamily cfNew : rowNew.getColumnFamilies())
         {
-            ColumnFamily cf = columnFamilies_.get(cfName);
+            ColumnFamily cf = columnFamilies_.get(cfNew.name());
             ColumnFamily cfDiff = null;
             if (cf == null)
-                rowDiff.getColumnFamilyMap().put(cfName, columnFamilies.get(cfName));
+                rowDiff.addColumnFamily(cfNew);
             else
             {
-                cfDiff = cf.diff(columnFamilies.get(cfName));
+                cfDiff = cf.diff(cfNew);
                 if (cfDiff != null)
-                    rowDiff.getColumnFamilyMap().put(cfName, cfDiff);
+                    rowDiff.addColumnFamily(cfDiff);
             }
         }
-        if (rowDiff.getColumnFamilyMap().size() != 0)
-            return rowDiff;
-        else
+        if (rowDiff.getColumnFamilies().isEmpty())
             return null;
+        else
+            return rowDiff;
     }
 
     public Row cloneMe()

Modified: incubator/cassandra/trunk/src/org/apache/cassandra/db/SuperColumn.java
URL: http://svn.apache.org/viewvc/incubator/cassandra/trunk/src/org/apache/cassandra/db/SuperColumn.java?rev=768108&r1=768107&r2=768108&view=diff
==============================================================================
--- incubator/cassandra/trunk/src/org/apache/cassandra/db/SuperColumn.java (original)
+++ incubator/cassandra/trunk/src/org/apache/cassandra/db/SuperColumn.java Fri Apr 24 00:46:19 2009
@@ -232,10 +232,9 @@
 
     public IColumn diff(IColumn column)
     {
-    	IColumn  columnDiff = new SuperColumn(column.name());
-    	Collection<IColumn> columns = column.getSubColumns();
+    	IColumn columnDiff = new SuperColumn(column.name());
 
-        for ( IColumn subColumn : columns )
+        for (IColumn subColumn : column.getSubColumns())
         {
         	IColumn columnInternal = columns_.get(subColumn.name());
         	if(columnInternal == null )
@@ -251,7 +250,8 @@
         		}
         	}
         }
-        if(columnDiff.getSubColumns().size() != 0)
+
+        if (!columnDiff.getSubColumns().isEmpty())
         	return columnDiff;
         else
         	return null;

Modified: incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadRepairManager.java
URL: http://svn.apache.org/viewvc/incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadRepairManager.java?rev=768108&r1=768107&r2=768108&view=diff
==============================================================================
--- incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadRepairManager.java (original)
+++ incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadRepairManager.java Fri Apr 24 00:46:19 2009
@@ -106,18 +106,12 @@
 	}
 
 	/*
-	 * This is the fn that should be used to scheule a read repair 
-	 * specify a endpoint on whcih the read repair should happen and the row mutaion
-	 * message that has the repaired row.
+	 * Schedules a read repair.
+	 * @param target endpoint on whcih the read repair should happen
+	 * @param rowMutationMessage the row mutation message that has the repaired row.
 	 */
 	public void schedule(EndPoint target, RowMutationMessage rowMutationMessage)
 	{
-        /*
-		Message message = new Message(StorageService.getLocalStorageEndPoint(),
-				StorageService.mutationStage_,
-				StorageService.readRepairVerbHandler_, new Object[]
-				{ rowMutationMessage });
-        */
         try
         {
             Message message = RowMutationMessage.makeRowMutationMessage(rowMutationMessage, StorageService.readRepairVerbHandler_);
@@ -126,7 +120,7 @@
         }
         catch ( IOException ex )
         {
-            logger_.info(LogUtil.throwableToString(ex));
+            logger_.error(LogUtil.throwableToString(ex));
         }
 	}
 }

Modified: incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadResponseResolver.java
URL: http://svn.apache.org/viewvc/incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadResponseResolver.java?rev=768108&r1=768107&r2=768108&view=diff
==============================================================================
--- incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadResponseResolver.java (original)
+++ incubator/cassandra/trunk/src/org/apache/cassandra/service/ReadResponseResolver.java Fri Apr 24 00:46:19 2009
@@ -126,35 +126,26 @@
 		{
 			retRow.repair(rowList.get(i));			
 		}
+
         // At  this point  we have the return row .
 		// Now we need to calculate the differnce 
 		// so that we can schedule read repairs 
-		
 		for (int i = 0 ; i < rowList.size(); i++)
 		{
-			// calculate the difference , since retRow is the resolved
-			// row it can be used as the super set , remember no deletes 
-			// will happen with diff its only for additions so far 
-			// TODO : handle deletes 
+			// since retRow is the resolved row it can be used as the super set
 			Row diffRow = rowList.get(i).diff(retRow);
 			if(diffRow == null) // no repair needs to happen
 				continue;
 			// create the row mutation message based on the diff and schedule a read repair 
 			RowMutation rowMutation = new RowMutation(table, key);            			
-	    	Map<String, ColumnFamily> columnFamilies = diffRow.getColumnFamilyMap();
-	        Set<String> cfNames = columnFamilies.keySet();
-	        
-	        for ( String cfName : cfNames )
+	        for (ColumnFamily cf : diffRow.getColumnFamilies())
 	        {
-	            ColumnFamily cf = columnFamilies.get(cfName);
 	            rowMutation.add(cf);
 	        }
             RowMutationMessage rowMutationMessage = new RowMutationMessage(rowMutation);
-	        // schedule the read repair
 	        ReadRepairManager.instance().schedule(endPoints.get(i),rowMutationMessage);
 		}
-        logger_.info("resolve: " + (System.currentTimeMillis() - startTime)
-                + " ms.");
+        logger_.info("resolve: " + (System.currentTimeMillis() - startTime) + " ms.");
 		return retRow;
 	}
 

Modified: incubator/cassandra/trunk/src/org/apache/cassandra/service/StorageProxy.java
URL: http://svn.apache.org/viewvc/incubator/cassandra/trunk/src/org/apache/cassandra/service/StorageProxy.java?rev=768108&r1=768107&r2=768108&view=diff
==============================================================================
--- incubator/cassandra/trunk/src/org/apache/cassandra/service/StorageProxy.java (original)
+++ incubator/cassandra/trunk/src/org/apache/cassandra/service/StorageProxy.java Fri Apr 24 00:46:19 2009
@@ -415,7 +415,7 @@
     private static Row strongRead(ReadCommand command) throws IOException, TimeoutException
     {
         // TODO: throw a thrift exception if we do not have N nodes
-
+        assert !command.isDigestQuery();
         ReadCommand readMessageDigestOnly = command.copy();
         readMessageDigestOnly.setDigestQuery(true);
 
@@ -464,7 +464,6 @@
                 QuorumResponseHandler<Row> quorumResponseHandlerRepair = new QuorumResponseHandler<Row>(
                         DatabaseDescriptor.getReplicationFactor(),
                         readResponseResolverRepair);
-                command.setDigestQuery(false);
                 logger_.info("DigestMismatchException: " + command.key);
                 Message messageRepair = command.makeReadMessage();
                 MessagingService.getMessagingInstance().sendRR(messageRepair, endPoints,