Skip to content

Commit

Permalink
also clean ups
Browse files Browse the repository at this point in the history
  • Loading branch information
tlrx committed Nov 30, 2021
1 parent 858d4c1 commit 34de0d0
Showing 1 changed file with 22 additions and 18 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -1298,9 +1298,11 @@ public void clusterStateProcessed(String source, ClusterState oldState, ClusterS
private final Set<SnapshotId> ongoingSnapshotsDeletions = ConcurrentCollections.newConcurrentSet();

/**
* Set of pending snapshots deletions whose deletion is conflicting with on-going restores, clones or repository statuses
* Set of pending snapshots deletions whose deletion is conflicting with on-going restores/clones/repository clean up or repository
* missing or read-only. Those sets are used to identify the cluster state updates to process in case they resolve some conflict.
*/
private final Set<SnapshotId> pendingDeletionsWithConflictingRestores = ConcurrentCollections.newConcurrentSet();
private final Set<SnapshotId> pendingDeletionsWithConflictingCleanUps = ConcurrentCollections.newConcurrentSet();
private final Set<SnapshotId> pendingDeletionsWithConflictingClones = ConcurrentCollections.newConcurrentSet();
private final Set<SnapshotId> pendingDeletionsWithConflictingRepos = ConcurrentCollections.newConcurrentSet();

Expand Down Expand Up @@ -1335,6 +1337,7 @@ private void triggerSnapshotsPendingDeletions(final ClusterChangedEvent event) {
}

final RepositoriesMetadata repositories = state.metadata().custom(RepositoriesMetadata.TYPE, RepositoriesMetadata.EMPTY);
final RepositoryCleanupInProgress cleanUps = state.custom(RepositoryCleanupInProgress.TYPE, RepositoryCleanupInProgress.EMPTY);

final Set<SnapshotId> currentDeletions = deletionsSources(state);
final Set<SnapshotId> currentRestores = restoreSources(state);
Expand All @@ -1360,14 +1363,8 @@ private void triggerSnapshotsPendingDeletions(final ClusterChangedEvent event) {
}
pendingDeletionsWithConflictingClones.remove(snapshotId);

if (currentDeletions.contains(snapshotId)) {
logger.trace("snapshot to delete [{}] is already queued", snapshotId);
pendingDeletionsWithConflictingRepos.remove(snapshotId);
continue;
}

if (state.custom(RepositoryCleanupInProgress.TYPE, RepositoryCleanupInProgress.EMPTY).hasCleanupInProgress()) {
if (pendingDeletionsWithConflictingRepos.add(snapshotId)) {
if (cleanUps.hasCleanupInProgress()) {
if (pendingDeletionsWithConflictingCleanUps.add(snapshotId)) {
logger.debug(
"a repository clean-up is in progress, cannot delete pending snapshot [{}] created at {}",
snapshotId,
Expand All @@ -1376,6 +1373,13 @@ private void triggerSnapshotsPendingDeletions(final ClusterChangedEvent event) {
}
continue;
}
pendingDeletionsWithConflictingCleanUps.remove(snapshotId);

if (currentDeletions.contains(snapshotId)) {
logger.trace("snapshot to delete [{}] is already queued", snapshotId);
pendingDeletionsWithConflictingRepos.remove(snapshotId);
continue;
}

final Optional<RepositoryMetadata> optionalRepository = findRepositoryForPendingDeletion(
repositories,
Expand Down Expand Up @@ -1424,6 +1428,7 @@ private void triggerSnapshotsPendingDeletions(final ClusterChangedEvent event) {

private void clearPendingDeletionsWithConflicts() {
pendingDeletionsWithConflictingRestores.clear();
pendingDeletionsWithConflictingCleanUps.clear();
pendingDeletionsWithConflictingClones.clear();
pendingDeletionsWithConflictingRepos.clear();
}
Expand Down Expand Up @@ -1458,21 +1463,20 @@ private boolean pendingDeletionsWithConflictsChanged(ClusterChangedEvent event)
}
}
if (pendingDeletionsWithConflictingRepos.isEmpty() == false) {
RepositoriesMetadata previous = event.previousState().metadata().custom(RepositoriesMetadata.TYPE, RepositoriesMetadata.EMPTY);
RepositoriesMetadata current = event.state().metadata().custom(RepositoriesMetadata.TYPE, RepositoriesMetadata.EMPTY);
if (previous.equals(current) == false) {
return true;
}
}
if (pendingDeletionsWithConflictingCleanUps.isEmpty() == false) {
boolean previousCleanUp = event.previousState()
.custom(RepositoryCleanupInProgress.TYPE, RepositoryCleanupInProgress.EMPTY)
.hasCleanupInProgress();
boolean currentCleanUp = event.state()
.custom(RepositoryCleanupInProgress.TYPE, RepositoryCleanupInProgress.EMPTY)
.hasCleanupInProgress();
if (previousCleanUp != currentCleanUp) {
return true;
}

RepositoriesMetadata previous = event.previousState().metadata().custom(RepositoriesMetadata.TYPE, RepositoriesMetadata.EMPTY);
RepositoriesMetadata current = event.state().metadata().custom(RepositoriesMetadata.TYPE, RepositoriesMetadata.EMPTY);
if (previous.equals(current) == false) {
return true;
}
return previousCleanUp != currentCleanUp;
}
return false;
}
Expand Down

0 comments on commit 34de0d0

Please sign in to comment.