You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by ma...@apache.org on 2016/02/23 11:16:56 UTC

[07/13] cassandra git commit: Only notify if the repaired status actually changed

Only notify if the repaired status actually changed

Patch by marcuse; reviewed by Branimir Lambov for CASSANDRA-11172


Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo
Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/67637d1b
Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/67637d1b
Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/67637d1b

Branch: refs/heads/trunk
Commit: 67637d1bbaf638162b479ed33a75e4ab3fb06e49
Parents: 14a2157
Author: Marcus Eriksson <ma...@apache.org>
Authored: Mon Feb 22 10:57:42 2016 +0100
Committer: Marcus Eriksson <ma...@apache.org>
Committed: Tue Feb 23 11:10:13 2016 +0100

----------------------------------------------------------------------
 CHANGES.txt                                             |  1 +
 .../cassandra/db/compaction/CompactionManager.java      | 12 +++++++++++-
 2 files changed, 12 insertions(+), 1 deletion(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/67637d1b/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index ef8c5a7..52bdcce 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 2.1.14
+ * Only notify if repair status changed (CASSANDRA-11172)
  * Add partition key to TombstoneOverwhelmingException error message (CASSANDRA-10888)
  * Use logback setting for 'cassandra -v' command (CASSANDRA-10767)
  * Fix sstableloader to unthrottle streaming by default (CASSANDRA-9714)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/67637d1b/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/compaction/CompactionManager.java b/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
index 55b873a..ec7cb45 100644
--- a/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
+++ b/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
@@ -24,6 +24,7 @@ import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Collection;
 import java.util.Collections;
+import java.util.HashMap;
 import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
@@ -472,7 +473,14 @@ public class CompactionManager implements CompactionManagerMBean
         logger.debug("Starting anticompaction for ranges {}", ranges);
         Set<SSTableReader> sstables = new HashSet<>(validatedForRepair);
         Set<SSTableReader> mutatedRepairStatuses = new HashSet<>();
+        // we should only notify that repair status changed if it actually did:
+        Set<SSTableReader> mutatedRepairStatusToNotify = new HashSet<>();
+        Map<SSTableReader, Boolean> wasRepairedBefore = new HashMap<>();
+        for (SSTableReader sstable : sstables)
+            wasRepairedBefore.put(sstable, sstable.isRepaired());
+
         Set<SSTableReader> nonAnticompacting = new HashSet<>();
+
         Iterator<SSTableReader> sstableIterator = sstables.iterator();
         try
         {
@@ -494,6 +502,8 @@ public class CompactionManager implements CompactionManagerMBean
                         sstable.descriptor.getMetadataSerializer().mutateRepairedAt(sstable.descriptor, repairedAt);
                         sstable.reloadSSTableMetadata();
                         mutatedRepairStatuses.add(sstable);
+                        if (!wasRepairedBefore.get(sstable))
+                            mutatedRepairStatusToNotify.add(sstable);
                         sstableIterator.remove();
                         shouldAnticompact = true;
                         break;
@@ -513,7 +523,7 @@ public class CompactionManager implements CompactionManagerMBean
                 }
             }
             validatedForRepair.release(Sets.union(nonAnticompacting, mutatedRepairStatuses));
-            cfs.getDataTracker().notifySSTableRepairedStatusChanged(mutatedRepairStatuses);
+            cfs.getDataTracker().notifySSTableRepairedStatusChanged(mutatedRepairStatusToNotify);
             cfs.getDataTracker().unmarkCompacting(Sets.union(nonAnticompacting, mutatedRepairStatuses));
             if (!sstables.isEmpty())
                 doAntiCompaction(cfs, ranges, sstables, repairedAt);