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/01/19 09:11:01 UTC

[04/15] cassandra git commit: Unmark sstables compacting once they are done in cleanup/scrub/upgradesstables.

Unmark sstables compacting once they are done in cleanup/scrub/upgradesstables.

Patch by marcuse; reviewed by Carl Yeksigian for CASSANDRA-10829


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

Branch: refs/heads/cassandra-3.0
Commit: 1b02a6eb40f4cd25675e9b003b4beff1a375ac2d
Parents: 58a0079
Author: Marcus Eriksson <ma...@apache.org>
Authored: Thu Dec 17 11:47:16 2015 +0100
Committer: Marcus Eriksson <ma...@apache.org>
Committed: Tue Jan 19 08:58:45 2016 +0100

----------------------------------------------------------------------
 CHANGES.txt                                     |  1 +
 .../apache/cassandra/db/ColumnFamilyStore.java  |  2 +-
 .../db/compaction/CompactionManager.java        | 24 ++++++++++++++------
 3 files changed, 19 insertions(+), 8 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/cassandra/blob/1b02a6eb/CHANGES.txt
----------------------------------------------------------------------
diff --git a/CHANGES.txt b/CHANGES.txt
index 3d84a30..9b1b79c 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 2.1.13
+ * Unmark sstables compacting once they are done in cleanup/scrub/upgradesstables (CASSANDRA-10829)
  * Revert CASSANDRA-10012 and add more logging (CASSANDRA-10961)
  * Allow simultaneous bootstrapping with strict consistency when no vnodes are used (CASSANDRA-11005)
  * Log a message when major compaction does not result in a single file (CASSANDRA-10847)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1b02a6eb/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
----------------------------------------------------------------------
diff --git a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
index a589b2e..a78f33f 100644
--- a/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
+++ b/src/java/org/apache/cassandra/db/ColumnFamilyStore.java
@@ -1921,7 +1921,7 @@ public class ColumnFamilyStore implements ColumnFamilyStoreMBean
             {
                 failingSince = System.nanoTime();
             }
-            else if (TimeUnit.MILLISECONDS.toNanos(100) > System.nanoTime() - failingSince)
+            else if (System.nanoTime() - failingSince > TimeUnit.MILLISECONDS.toNanos(100))
             {
                 List<SSTableReader> released = new ArrayList<>();
                 for (SSTableReader reader : view.sstables)

http://git-wip-us.apache.org/repos/asf/cassandra/blob/1b02a6eb/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 9e71ca0..55b873a 100644
--- a/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
+++ b/src/java/org/apache/cassandra/db/compaction/CompactionManager.java
@@ -283,11 +283,14 @@ public class CompactionManager implements CompactionManagerMBean
             logger.info("No sstables for {}.{}", cfs.keyspace.getName(), cfs.name);
             return AllSSTableOpStatus.SUCCESSFUL;
         }
+        Set<SSTableReader> sstables = Sets.newHashSet(operation.filterSSTables(compactingSSTables));
+        Set<SSTableReader> filteredAway = Sets.difference(Sets.newHashSet(compactingSSTables), sstables);
+        cfs.getDataTracker().unmarkCompacting(filteredAway);
+        final Set<SSTableReader> finished = Sets.newConcurrentHashSet();
+
+        List<Future<Object>> futures = new ArrayList<>();
         try
         {
-            Iterable<SSTableReader> sstables = operation.filterSSTables(compactingSSTables);
-            List<Future<Object>> futures = new ArrayList<>();
-
             for (final SSTableReader sstable : sstables)
             {
                 if (executor.isShutdown())
@@ -295,23 +298,29 @@ public class CompactionManager implements CompactionManagerMBean
                     logger.info("Executor has shut down, not submitting task");
                     return AllSSTableOpStatus.ABORTED;
                 }
-
                 futures.add(executor.submit(new Callable<Object>()
                 {
                     @Override
                     public Object call() throws Exception
                     {
-                        operation.execute(sstable);
+                        try
+                        {
+                            operation.execute(sstable);
+                        }
+                        finally
+                        {
+                            cfs.getDataTracker().unmarkCompacting(Collections.singleton(sstable));
+                            finished.add(sstable);
+                        }
                         return this;
                     }
                 }));
             }
-
             FBUtilities.waitOnFutures(futures);
         }
         finally
         {
-            cfs.getDataTracker().unmarkCompacting(compactingSSTables);
+            cfs.getDataTracker().unmarkCompacting(Sets.difference(sstables, finished));
         }
         return AllSSTableOpStatus.SUCCESSFUL;
     }
@@ -1144,6 +1153,7 @@ public class CompactionManager implements CompactionManagerMBean
                 anticompactedSSTables.addAll(repairedSSTableWriter.finish(repairedAt));
                 anticompactedSSTables.addAll(unRepairedSSTableWriter.finish(ActiveRepairService.UNREPAIRED_SSTABLE));
                 successfullyAntiCompactedSSTables.add(sstable);
+                cfs.getDataTracker().unmarkCompacting(sstableAsSet);
             }
             catch (Throwable e)
             {