Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

WAL-only cleanup #589

Open
wants to merge 1 commit into
base: palantir-cassandra-2.2.18
Choose a base branch
from
Open
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
10 changes: 0 additions & 10 deletions src/java/com/palantir/cassandra/db/ColumnFamilyStoreManager.java
Original file line number Diff line number Diff line change
Expand Up @@ -62,16 +62,6 @@ public Map<Descriptor, Set<Integer>> filterValidAncestors(CFMetaData cfMetaData,
return validator.filterValidAncestors(cfMetaData, sstableToCompletedAncestors, unfinishedCompactions);
}

@Override
public boolean shouldRemoveUnusedSstablesBasedOnAncestorMetadata() {
return validator.shouldRemoveUnusedSstablesBasedOnAncestorMetadata();
}

@Override
public boolean shouldSkipAncestorCleanupBasedOnAncestorMetadata() {
return validator.shouldSkipAncestorCleanupBasedOnAncestorMetadata();
}

@Override
public void markForDeletion(CFMetaData cfMetaData, Set<Descriptor> descriptors)
{
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,22 +33,4 @@ public interface IColumnFamilyStoreValidator
*/
Map<Descriptor, Set<Integer>> filterValidAncestors(CFMetaData cfMetaData,
Map<Descriptor, Set<Integer>> sstableToCompletedAncestors, Map<Integer, UUID> unfinishedCompactions);

/**
* @return true if Cassandra should use ancestry metdata to cleanup unused SSTables on startup by running
* {@link org.apache.cassandra.db.ColumnFamilyStore#removeUnusedSstables(CFMetaData, Map)}, false otherwise
* (e.g. if a different cleanup system is being used outside of {@link org.apache.cassandra.service.CassandraDaemon}).
*/
default boolean shouldRemoveUnusedSstablesBasedOnAncestorMetadata() {
return true;
}

/**
* @return true if Cassandra should skip cleaning up ancestors during
* {@link org.apache.cassandra.db.ColumnFamilyStore#removeUnusedSstables(CFMetaData, Map)}, false otherwise. Note
* that this flag does not control whether compaction products being cleaned up.
*/
default boolean shouldSkipAncestorCleanupBasedOnAncestorMetadata() {
return false;
}
}
132 changes: 0 additions & 132 deletions src/java/org/apache/cassandra/db/ColumnFamilyStore.java
Original file line number Diff line number Diff line change
Expand Up @@ -665,138 +665,6 @@ public boolean accept(File pathname)
}
}

/**
* Replacing compacted sstables is atomic as far as observers of Tracker are concerned, but not on the
* filesystem: first the new sstables are renamed to "live" status (i.e., the tmp marker is removed), then
* their ancestors are removed.
*
* If an unclean shutdown happens at the right time, we can thus end up with both the new ones and their
* ancestors "live" in the system. This is harmless for normal data, but for counters it can cause overcounts.
*
* To prevent this, we record sstables being compacted in the system keyspace. If we find unfinished
* compactions, we remove the new ones (since those may be incomplete -- under LCS, we may create multiple
* sstables from any given ancestor).
*/
public static void removeUnusedSstables(CFMetaData metadata, Map<Integer, UUID> unfinishedCompactions)
{
Directories directories = new Directories(metadata);
Set<Integer> allGenerations = new HashSet<>();
for (Descriptor desc : directories.sstableLister().list().keySet())
allGenerations.add(desc.generation);

// sanity-check unfinishedCompactions
Set<Integer> unfinishedGenerations = unfinishedCompactions.keySet();
if (!allGenerations.containsAll(unfinishedGenerations))
{
HashSet<Integer> missingGenerations = new HashSet<>(unfinishedGenerations);
missingGenerations.removeAll(allGenerations);
logger.info("Unfinished compactions reference missing sstables of generations",
SafeArg.of("keyspace", metadata.ksName), SafeArg.of("cf", metadata.cfName),
SafeArg.of("missingGenerations", missingGenerations));
}

// remove new sstables from compactions that didn't complete, and compute
// set of ancestors that shouldn't exist anymore
Map<Descriptor, Set<Integer>> allSstableToAncestors = new HashMap<>();
Set<Integer> completedAncestors = new HashSet<>();
Map<Descriptor, Set<Component>> allNonTempSstableFiles = directories.sstableLister().skipTemporary(true).list();
for (Map.Entry<Descriptor, Set<Component>> sstableFiles : allNonTempSstableFiles.entrySet())
{
// we rename the Data component last - if it does not exist as a final file, we should ignore this sstable and
// it will be removed during startup
if (!sstableFiles.getValue().contains(Component.DATA))
continue;

Descriptor desc = sstableFiles.getKey();

Set<Integer> ancestors;
try
{
CompactionMetadata compactionMetadata = (CompactionMetadata) desc.getMetadataSerializer().deserialize(desc, MetadataType.COMPACTION);
ancestors = compactionMetadata.ancestors;
}
catch (IOException e)
{
throw new FSReadError(e, desc.filenameFor(Component.STATS));
}
catch (NullPointerException e)
{
throw new FSReadError(e, "Failed to remove unfinished compaction leftovers (file: " + desc.filenameFor(Component.STATS) + "). See log for details.");
}
allSstableToAncestors.put(desc, ancestors);
}

allSstableToAncestors = ColumnFamilyStoreManager.instance.filterValidAncestors(metadata, allSstableToAncestors, unfinishedCompactions);
SafeArg<Map<Integer, Set<Integer>>> ancestorsArg = SafeArg.of(
"sstableToAncestors",
allSstableToAncestors.entrySet().stream()
.collect(Collectors.toMap(
(Map.Entry<Descriptor, Set<Integer>> e) -> e.getKey().generation,
Map.Entry::getValue)));

Set<UUID> cleanedUnfinishedCompactions = new HashSet<>();
for (Map.Entry<Descriptor, Set<Integer>> sstableToAncestors : allSstableToAncestors.entrySet())
{
Descriptor desc = sstableToAncestors.getKey();
Set<Integer> ancestors = sstableToAncestors.getValue();
if (!ancestors.isEmpty()
&& unfinishedGenerations.containsAll(ancestors)
&& allGenerations.containsAll(ancestors))
{
// any of the ancestors would work, so we'll just lookup the compaction task ID with the first one
UUID compactionTaskID = unfinishedCompactions.get(ancestors.iterator().next());
assert compactionTaskID != null;
if (DISABLE_COMPACTION_PRODUCT_CLEANUP)
{
logger.info("Would have deleted unfinished compaction product", UnsafeArg.of("desc", desc),
SafeArg.of("keyspace", desc.ksname), SafeArg.of("cf", desc.cfname),
SafeArg.of("generation", desc.generation), ancestorsArg);
}
else
{
logger.info("Going to delete unfinished compaction product", UnsafeArg.of("desc", desc),
SafeArg.of("keyspace", desc.ksname), SafeArg.of("cf", desc.cfname),
SafeArg.of("generation", desc.generation), ancestorsArg);
SSTable.delete(desc, allNonTempSstableFiles.get(desc));
}
cleanedUnfinishedCompactions.add(compactionTaskID);
}
else
{
completedAncestors.addAll(ancestors);
}
}
cleanedUnfinishedCompactions.forEach(SystemKeyspace::finishCompaction);

if (ColumnFamilyStoreManager.instance.shouldSkipAncestorCleanupBasedOnAncestorMetadata()) {
return;
}

// remove old sstables from compactions that did complete
for (Map.Entry<Descriptor, Set<Component>> sstableFiles : directories.sstableLister().list().entrySet())
{
Descriptor desc = sstableFiles.getKey();
if (completedAncestors.contains(desc.generation))
{
if (DRY_RUN_NON_COMPACTING_UNUSED_SSTABLE_CLEANUP && unfinishedCompactions.isEmpty())
{
logger.warn("Would have deleted leftover compaction ancestor", UnsafeArg.of("desc", desc),
SafeArg.of("keyspace", desc.ksname), SafeArg.of("cf", desc.cfname),
SafeArg.of("generation", desc.generation), ancestorsArg);
} else
{
// if any of the ancestors were participating in a compaction, finish that compaction
logger.warn("Going to delete leftover compaction ancestor", UnsafeArg.of("desc", desc),
SafeArg.of("keyspace", desc.ksname), SafeArg.of("cf", desc.cfname),
SafeArg.of("generation", desc.generation), ancestorsArg);
SSTable.delete(desc, sstableFiles.getValue());
Optional.ofNullable(unfinishedCompactions.get(desc.generation))
.ifPresent(SystemKeyspace::finishCompaction);
}
}
}
}

/**
* See #{@code StorageService.loadNewSSTables(String, String)} for more info
*
Expand Down
5 changes: 0 additions & 5 deletions src/java/org/apache/cassandra/service/CassandraDaemon.java
Original file line number Diff line number Diff line change
Expand Up @@ -284,11 +284,6 @@ private void completeSetupMayThrowSstableException() {
Map<Pair<String, String>, Map<Integer, UUID>> unfinishedCompactions = SystemKeyspace.getUnfinishedCompactions();
for (String keyspaceName : Schema.instance.getKeyspaces())
{
for (CFMetaData cfm : Schema.instance.getKeyspaceMetaData(keyspaceName).values())
{
ColumnFamilyStore.removeUnusedSstables(cfm, unfinishedCompactions.getOrDefault(cfm.ksAndCFName, ImmutableMap.of()));
}

if (keyspaceName.equals(SystemKeyspace.NAME))
continue;

Expand Down
Loading