Skip to content

Commit

Permalink
Showing 15 changed files with 1,126 additions and 641 deletions.
380 changes: 26 additions & 354 deletions gcs/src/main/java/com/google/cloud/hadoop/fs/gcs/CoopLockFsck.java

Large diffs are not rendered by default.

Large diffs are not rendered by default.

Original file line number Diff line number Diff line change
@@ -18,6 +18,8 @@

import static com.google.cloud.hadoop.fs.gcs.GoogleHadoopFileSystemBase.AUTHENTICATION_PREFIX;
import static com.google.cloud.hadoop.fs.gcs.GoogleHadoopFileSystemConfiguration.GCS_COOPERATIVE_LOCKING_EXPIRATION_TIMEOUT_MS;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.DELETE;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.RENAME;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockRecordsDao.LOCK_DIRECTORY;
import static com.google.cloud.hadoop.util.EntriesCredentialConfiguration.ENABLE_SERVICE_ACCOUNTS_SUFFIX;
import static com.google.cloud.hadoop.util.EntriesCredentialConfiguration.SERVICE_ACCOUNT_EMAIL_SUFFIX;
@@ -61,7 +63,7 @@ public class CoopLockRepairIntegrationTest {

private static final Gson GSON = new Gson();

private static final String OPERATION_FILENAME_PATTERN =
private static final String OPERATION_FILENAME_PATTERN_FORMAT =
"[0-9]{8}T[0-9]{6}\\.[0-9]{3}Z_%s_[a-z0-9\\-]+";

private static GoogleCloudStorageOptions gcsOptions;
@@ -175,10 +177,9 @@ private void moveDirectoryOperationRepairedAfterFailedCopy(String command) throw
.collect(toList());

assertThat(lockFiles).hasSize(2);
URI lockFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.lock").get();
URI logFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.log").get();
String filenamePattern = String.format(OPERATION_FILENAME_PATTERN_FORMAT, RENAME);
URI lockFileUri = matchFile(lockFiles, filenamePattern + "\\.lock").get();
URI logFileUri = matchFile(lockFiles, filenamePattern + "\\.log").get();

String lockContent = gcsfsIHelper.readTextFile(bucketName, lockFileUri.getPath());
assertThat(GSON.fromJson(lockContent, RenameOperation.class).setLockEpochSeconds(0))
@@ -248,10 +249,9 @@ public void moveDirectoryOperationRepairedAfterFailedDelete() throws Exception {
.collect(toList());

assertThat(lockFiles).hasSize(2);
URI lockFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.lock").get();
URI logFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.log").get();
String filenameFormat = String.format(OPERATION_FILENAME_PATTERN_FORMAT, RENAME);
URI lockFileUri = matchFile(lockFiles, filenameFormat + "\\.lock").get();
URI logFileUri = matchFile(lockFiles, filenameFormat + "\\.log").get();

String lockContent = gcsfsIHelper.readTextFile(bucketName, lockFileUri.getPath());
assertThat(GSON.fromJson(lockContent, RenameOperation.class).setLockEpochSeconds(0))
@@ -316,11 +316,14 @@ public void deleteDirectoryOperationRolledForward() throws Exception {
.collect(toList());

assertThat(lockFiles).hasSize(2);
URI lockFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "delete") + "\\.lock").get();
String filenamePattern = String.format(OPERATION_FILENAME_PATTERN_FORMAT, DELETE);
URI lockFileUri = matchFile(lockFiles, filenamePattern + "\\.lock").get();
URI logFileUri = matchFile(lockFiles, filenamePattern + "\\.log").get();
String lockContent = gcsfsIHelper.readTextFile(bucketName, lockFileUri.getPath());
assertThat(GSON.fromJson(lockContent, DeleteOperation.class).setLockEpochSeconds(0))
.isEqualTo(new DeleteOperation().setLockEpochSeconds(0).setResource(dirUri.toString()));
assertThat(gcsfsIHelper.readTextFile(bucketName, logFileUri.getPath()))
.isEqualTo(dirUri.resolve(fileName) + "\n" + dirUri + "\n");
}

private Configuration getTestConfiguration() {
Original file line number Diff line number Diff line change
@@ -254,7 +254,7 @@ public GoogleCloudStorageItemInfo composeObjects(
*
* @return the {@link GoogleCloudStorage} objected wrapped by this class.
*/
protected GoogleCloudStorage getDelegate() {
public GoogleCloudStorage getDelegate() {
return delegate;
}
}
Original file line number Diff line number Diff line change
@@ -29,8 +29,8 @@
import com.google.api.client.util.Clock;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorage.ListPage;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorageFileSystemOptions.TimestampUpdatePredicate;
import com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationDao;
import com.google.cloud.hadoop.gcsio.cooplock.CoopLockRecordsDao;
import com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationDelete;
import com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationRename;
import com.google.cloud.hadoop.util.LazyExecutorService;
import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.CharMatcher;
@@ -50,17 +50,16 @@
import java.nio.channels.WritableByteChannel;
import java.nio.file.DirectoryNotEmptyException;
import java.nio.file.FileAlreadyExistsException;
import java.time.Instant;
import java.util.ArrayList;
import java.util.Collections;
import java.util.Comparator;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import java.util.TreeMap;
import java.util.UUID;
import java.util.concurrent.ExecutionException;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
@@ -110,9 +109,6 @@ public class GoogleCloudStorageFileSystem {

private final PathCodec pathCodec;

private final CoopLockRecordsDao coopLockRecordsDao;
private final CoopLockOperationDao coopLockOperationDao;

// Executor for updating directory timestamps.
private ExecutorService updateTimestampsExecutor = createUpdateTimestampsExecutor();

@@ -165,15 +161,10 @@ public GoogleCloudStorageFileSystem(

checkArgument(credential != null, "credential must not be null");

GoogleCloudStorageImpl gcsImpl =
new GoogleCloudStorageImpl(options.getCloudStorageOptions(), credential);
this.gcs = gcsImpl;
this.gcs = new GoogleCloudStorageImpl(options.getCloudStorageOptions(), credential);
this.options = options;
this.pathCodec = options.getPathCodec();

this.coopLockRecordsDao = new CoopLockRecordsDao(gcsImpl);
this.coopLockOperationDao = new CoopLockOperationDao(gcsImpl, pathCodec);

if (options.isPerformanceCacheEnabled()) {
this.gcs =
new PerformanceCachingGoogleCloudStorage(this.gcs, options.getPerformanceCacheOptions());
@@ -203,26 +194,6 @@ public GoogleCloudStorageFileSystem(
this.gcs = gcs;
this.options = options;
this.pathCodec = options.getPathCodec();
this.coopLockRecordsDao = null;
this.coopLockOperationDao = null;
}

/**
* Constructs a GoogleCloudStorageFilesystem based on an already-configured underlying
* GoogleCloudStorageImpl {@code gcs}. Any options pertaining to GCS creation will be ignored.
*/
@VisibleForTesting
public GoogleCloudStorageFileSystem(
GoogleCloudStorageImpl gcs, GoogleCloudStorageFileSystemOptions options) {
this.gcs = gcs;
this.options = options;
this.pathCodec = options.getPathCodec();
this.coopLockRecordsDao = new CoopLockRecordsDao(gcs);
this.coopLockOperationDao = new CoopLockOperationDao(gcs, pathCodec);
}

public CoopLockRecordsDao getCoopLockRecordsDao() {
return coopLockRecordsDao;
}

@VisibleForTesting
@@ -417,9 +388,13 @@ public void delete(URI path, boolean recursive) throws IOException {
() -> getFileInfoInternal(parentId, /* inferImplicitDirectories= */ false));
}

List<FileInfo> itemsToDelete = new ArrayList<>();
List<FileInfo> bucketsToDelete = new ArrayList<>();
Optional<CoopLockOperationDelete> coopLockOp =
options.enableCooperativeLocking() && fileInfo.isDirectory()
? Optional.of(CoopLockOperationDelete.create(gcs, pathCodec, fileInfo.getPath()))
: Optional.empty();
coopLockOp.ifPresent(CoopLockOperationDelete::lock);

List<FileInfo> itemsToDelete;
// Delete sub-items if it is a directory.
if (fileInfo.isDirectory()) {
itemsToDelete =
@@ -429,32 +404,20 @@ public void delete(URI path, boolean recursive) throws IOException {
if (!itemsToDelete.isEmpty() && !recursive) {
throw new DirectoryNotEmptyException("Cannot delete a non-empty directory.");
}
}

if (fileInfo.getItemInfo().isBucket()) {
bucketsToDelete.add(fileInfo);
} else {
itemsToDelete.add(fileInfo);
itemsToDelete = new ArrayList<>();
}

if (options.enableCooperativeLocking() && fileInfo.isDirectory()) {
String operationId = UUID.randomUUID().toString();
StorageResourceId resourceId = pathCodec.validatePathAndGetId(fileInfo.getPath(), true);

coopLockRecordsDao.lockPaths(operationId, resourceId);
Future<?> lockUpdateFuture = Futures.immediateCancelledFuture();
try {
lockUpdateFuture =
coopLockOperationDao.persistDeleteOperation(
path, itemsToDelete, bucketsToDelete, operationId, resourceId, lockUpdateFuture);
List<FileInfo> bucketsToDelete = new ArrayList<>();
(fileInfo.getItemInfo().isBucket() ? bucketsToDelete : itemsToDelete).add(fileInfo);

deleteInternal(itemsToDelete, bucketsToDelete);
coopLockRecordsDao.unlockPaths(operationId, resourceId);
} finally {
lockUpdateFuture.cancel(/* mayInterruptIfRunning= */ true);
}
} else {
coopLockOp.ifPresent(o -> o.persistAndScheduleRenewal(itemsToDelete, bucketsToDelete));
try {
deleteInternal(itemsToDelete, bucketsToDelete);

coopLockOp.ifPresent(CoopLockOperationDelete::unlock);
} finally {
coopLockOp.ifPresent(CoopLockOperationDelete::cancelRenewal);
}

repairImplicitDirectory(parentInfoFuture);
@@ -786,16 +749,8 @@ public void compose(List<URI> sources, URI destination, String contentType) thro
* copied and not the content of any file.
*/
private void renameInternal(FileInfo srcInfo, URI dst) throws IOException {
if (srcInfo.isDirectory() && options.enableCooperativeLocking()) {
String operationId = UUID.randomUUID().toString();
StorageResourceId srcResourceId = pathCodec.validatePathAndGetId(srcInfo.getPath(), true);
StorageResourceId dstResourceId = pathCodec.validatePathAndGetId(dst, true);

coopLockRecordsDao.lockPaths(operationId, srcResourceId, dstResourceId);
renameDirectoryInternal(srcInfo, dst, operationId);
coopLockRecordsDao.unlockPaths(operationId, srcResourceId, dstResourceId);
} else if (srcInfo.isDirectory()) {
renameDirectoryInternal(srcInfo, dst, /* operationId= */ null);
if (srcInfo.isDirectory()) {
renameDirectoryInternal(srcInfo, dst);
} else {
URI src = srcInfo.getPath();
StorageResourceId srcResourceId = pathCodec.validatePathAndGetId(src, true);
@@ -825,11 +780,16 @@ private void renameInternal(FileInfo srcInfo, URI dst) throws IOException {
*
* @see #renameInternal
*/
private void renameDirectoryInternal(FileInfo srcInfo, URI dst, String operationId)
throws IOException {
private void renameDirectoryInternal(FileInfo srcInfo, URI dst) throws IOException {
checkArgument(srcInfo.isDirectory(), "'%s' should be a directory", srcInfo);

Pattern markerFilePattern = options.getMarkerFilePattern();
URI src = srcInfo.getPath();

Optional<CoopLockOperationRename> coopLockOp =
options.enableCooperativeLocking() && src.getAuthority().equals(dst.getAuthority())
? Optional.of(CoopLockOperationRename.create(gcs, pathCodec, src, dst))
: Optional.empty();
coopLockOp.ifPresent(CoopLockOperationRename::lock);

// Mapping from each src to its respective dst.
// Sort src items so that parent directories appear before their children.
@@ -839,13 +799,14 @@ private void renameDirectoryInternal(FileInfo srcInfo, URI dst, String operation

// List of individual paths to rename;
// we will try to carry out the copies in this list's order.
List<FileInfo> srcItemInfos = listAllFileInfoForPrefix(srcInfo.getPath());
List<FileInfo> srcItemInfos = listAllFileInfoForPrefix(src);

// Convert to the destination directory.
dst = FileInfo.convertToDirectoryPath(pathCodec, dst);

// Create a list of sub-items to copy.
String prefix = srcInfo.getPath().toString();
Pattern markerFilePattern = options.getMarkerFilePattern();
String prefix = src.toString();
for (FileInfo srcItemInfo : srcItemInfos) {
String relativeItemName = srcItemInfo.getPath().toString().substring(prefix.length());
URI dstItemName = dst.resolve(relativeItemName);
@@ -856,69 +817,56 @@ private void renameDirectoryInternal(FileInfo srcInfo, URI dst, String operation
}
}

Future<?> lockUpdateFuture = null;
Instant operationInstant = Instant.now();
if (options.enableCooperativeLocking()
&& srcInfo.getItemInfo().getBucketName().equals(dst.getAuthority())) {
lockUpdateFuture =
coopLockOperationDao.persistUpdateOperation(
srcInfo,
dst,
operationId,
srcToDstItemNames,
srcToDstMarkerItemNames,
operationInstant);
}

// Create the destination directory.
mkdir(dst);
coopLockOp.ifPresent(
o -> o.persistAndScheduleRenewal(srcToDstItemNames, srcToDstMarkerItemNames));
try {
// Create the destination directory.
mkdir(dst);

// First, copy all items except marker items
copyInternal(srcToDstItemNames);
// Finally, copy marker items (if any) to mark rename operation success
copyInternal(srcToDstMarkerItemNames);
// First, copy all items except marker items
copyInternal(srcToDstItemNames);
// Finally, copy marker items (if any) to mark rename operation success
copyInternal(srcToDstMarkerItemNames);

if (options.enableCooperativeLocking()
&& srcInfo.getItemInfo().getBucketName().equals(dst.getAuthority())) {
coopLockOperationDao.checkpointUpdateOperation(srcInfo, dst, operationId, operationInstant);
}
coopLockOp.ifPresent(CoopLockOperationRename::checkpoint);

// So far, only the destination directories are updated. Only do those now:
if (!srcToDstItemNames.isEmpty() || !srcToDstMarkerItemNames.isEmpty()) {
List<URI> allDestinationUris =
new ArrayList<>(srcToDstItemNames.size() + srcToDstMarkerItemNames.size());
allDestinationUris.addAll(srcToDstItemNames.values());
allDestinationUris.addAll(srcToDstMarkerItemNames.values());
// So far, only the destination directories are updated. Only do those now:
if (!srcToDstItemNames.isEmpty() || !srcToDstMarkerItemNames.isEmpty()) {
List<URI> allDestinationUris =
new ArrayList<>(srcToDstItemNames.size() + srcToDstMarkerItemNames.size());
allDestinationUris.addAll(srcToDstItemNames.values());
allDestinationUris.addAll(srcToDstMarkerItemNames.values());

tryUpdateTimestampsForParentDirectories(allDestinationUris, allDestinationUris);
}

List<FileInfo> bucketsToDelete = new ArrayList<>(1);
List<FileInfo> srcItemsToDelete = new ArrayList<>(srcToDstItemNames.size() + 1);
srcItemsToDelete.addAll(srcToDstItemNames.keySet());
if (srcInfo.getItemInfo().isBucket()) {
bucketsToDelete.add(srcInfo);
} else {
// If src is a directory then srcItemInfos does not contain its own name,
// therefore add it to the list before we delete items in the list.
srcItemsToDelete.add(srcInfo);
}
tryUpdateTimestampsForParentDirectories(allDestinationUris, allDestinationUris);
}

// First delete marker files from the src
deleteInternal(new ArrayList<>(srcToDstMarkerItemNames.keySet()), new ArrayList<>());
// Then delete rest of the items that we successfully copied.
deleteInternal(srcItemsToDelete, bucketsToDelete);
List<FileInfo> bucketsToDelete = new ArrayList<>(1);
List<FileInfo> srcItemsToDelete = new ArrayList<>(srcToDstItemNames.size() + 1);
srcItemsToDelete.addAll(srcToDstItemNames.keySet());
if (srcInfo.getItemInfo().isBucket()) {
bucketsToDelete.add(srcInfo);
} else {
// If src is a directory then srcItemInfos does not contain its own name,
// therefore add it to the list before we delete items in the list.
srcItemsToDelete.add(srcInfo);
}

// if we deleted a bucket, then there no need to update timestamps
if (bucketsToDelete.isEmpty()) {
List<URI> srcItemNames =
srcItemInfos.stream().map(FileInfo::getPath).collect(toCollection(ArrayList::new));
// Any path that was deleted, we should update the parent except for parents we also deleted
tryUpdateTimestampsForParentDirectories(srcItemNames, srcItemNames);
}
// First delete marker files from the src
deleteInternal(new ArrayList<>(srcToDstMarkerItemNames.keySet()), new ArrayList<>());
// Then delete rest of the items that we successfully copied.
deleteInternal(srcItemsToDelete, bucketsToDelete);

// if we deleted a bucket, then there no need to update timestamps
if (bucketsToDelete.isEmpty()) {
List<URI> srcItemNames =
srcItemInfos.stream().map(FileInfo::getPath).collect(toCollection(ArrayList::new));
// Any path that was deleted, we should update the parent except for parents we also deleted
tryUpdateTimestampsForParentDirectories(srcItemNames, srcItemNames);
}

if (lockUpdateFuture != null) {
lockUpdateFuture.cancel(/* mayInterruptIfRunning= */ false);
coopLockOp.ifPresent(CoopLockOperationRename::unlock);
} finally {
coopLockOp.ifPresent(CoopLockOperationRename::cancelRenewal);
}
}

Original file line number Diff line number Diff line change
@@ -1,9 +1,24 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import static com.google.cloud.hadoop.gcsio.CreateObjectOptions.DEFAULT_CONTENT_TYPE;
import static com.google.cloud.hadoop.gcsio.CreateObjectOptions.EMPTY_METADATA;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockRecordsDao.LOCK_DIRECTORY;
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Preconditions.checkState;
import static com.google.common.collect.ImmutableList.toImmutableList;
import static com.google.common.util.concurrent.Uninterruptibles.sleepUninterruptibly;
@@ -16,9 +31,9 @@
import com.google.cloud.hadoop.gcsio.PathCodec;
import com.google.cloud.hadoop.gcsio.StorageResourceId;
import com.google.common.collect.ImmutableList;
import com.google.common.collect.ImmutableSet;
import com.google.common.collect.Streams;
import com.google.common.flogger.GoogleLogger;
import com.google.common.util.concurrent.ThreadFactoryBuilder;
import com.google.gson.Gson;
import java.io.BufferedReader;
import java.io.IOException;
@@ -31,7 +46,6 @@
import java.time.format.DateTimeFormatter;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.concurrent.Executors;
import java.util.concurrent.Future;
import java.util.concurrent.ScheduledExecutorService;
@@ -43,9 +57,11 @@
public class CoopLockOperationDao {
private static final GoogleLogger logger = GoogleLogger.forEnclosingClass();

private static final Set<String> VALID_OPERATIONS = ImmutableSet.of("delete", "rename");
public static final String RENAME_LOG_RECORD_SEPARATOR = " -> ";

private static final String OPERATION_LOG_FILE_FORMAT = "%s_%s_%s.log";
private static final String OPERATION_LOCK_FILE_FORMAT = "%s_%s_%s.lock";

private static final CreateObjectOptions CREATE_OBJECT_OPTIONS =
new CreateObjectOptions(/* overwriteExisting= */ false, "application/text", EMPTY_METADATA);
private static final CreateObjectOptions UPDATE_OBJECT_OPTIONS =
@@ -56,6 +72,11 @@ public class CoopLockOperationDao {

private static final Gson GSON = new Gson();

private final ScheduledExecutorService scheduledThreadPool =
Executors.newScheduledThreadPool(
/* corePoolSize= */ 0,
new ThreadFactoryBuilder().setNameFormat("coop-lock-thread-%d").setDaemon(true).build());

private GoogleCloudStorage gcs;
private PathCodec pathCodec;

@@ -65,20 +86,18 @@ public CoopLockOperationDao(GoogleCloudStorage gcs, PathCodec pathCodec) {
}

public Future<?> persistDeleteOperation(
URI path,
List<FileInfo> itemsToDelete,
List<FileInfo> bucketsToDelete,
String operationId,
Instant operationInstant,
StorageResourceId resourceId,
Future<?> lockUpdateFuture)
List<FileInfo> itemsToDelete,
List<FileInfo> bucketsToDelete)
throws IOException {
Instant operationInstant = Instant.now();
URI operationLockPath =
writeOperationFile(
path.getAuthority(),
resourceId.getBucketName(),
OPERATION_LOCK_FILE_FORMAT,
CREATE_OBJECT_OPTIONS,
"delete",
CoopLockOperationType.DELETE,
operationId,
operationInstant,
ImmutableList.of(
@@ -91,86 +110,90 @@ public Future<?> persistDeleteOperation(
.map(i -> i.getItemInfo().getResourceId().toString())
.collect(toImmutableList());
writeOperationFile(
path.getAuthority(),
resourceId.getBucketName(),
OPERATION_LOG_FILE_FORMAT,
CREATE_OBJECT_OPTIONS,
"delete",
CoopLockOperationType.DELETE,
operationId,
operationInstant,
logRecords);
// Schedule lock expiration update
lockUpdateFuture =
scheduleLockUpdate(
operationId,
operationLockPath,
DeleteOperation.class,
(o, i) -> o.setLockEpochSeconds(i.getEpochSecond()));
return lockUpdateFuture;
return scheduleLockUpdate(
operationId,
operationLockPath,
DeleteOperation.class,
(o, i) -> o.setLockEpochSeconds(i.getEpochSecond()));
}

public Future<?> persistUpdateOperation(
FileInfo srcInfo,
URI dst,
public Future<?> persistRenameOperation(
String operationId,
Instant operationInstant,
StorageResourceId src,
StorageResourceId dst,
Map<FileInfo, URI> srcToDstItemNames,
Map<FileInfo, URI> srcToDstMarkerItemNames,
Instant operationInstant)
Map<FileInfo, URI> srcToDstMarkerItemNames)
throws IOException {
Future<?> lockUpdateFuture;
URI operationLockPath =
writeOperationFile(
dst.getAuthority(),
dst.getBucketName(),
OPERATION_LOCK_FILE_FORMAT,
CREATE_OBJECT_OPTIONS,
"rename",
CoopLockOperationType.RENAME,
operationId,
operationInstant,
ImmutableList.of(
GSON.toJson(
new RenameOperation()
.setLockEpochSeconds(operationInstant.getEpochSecond())
.setSrcResource(srcInfo.getPath().toString())
.setSrcResource(src.toString())
.setDstResource(dst.toString())
.setCopySucceeded(false))));
List<String> logRecords =
Streams.concat(
srcToDstItemNames.entrySet().stream(), srcToDstMarkerItemNames.entrySet().stream())
.map(e -> e.getKey().getItemInfo().getResourceId() + " -> " + e.getValue())
.map(
e ->
e.getKey().getItemInfo().getResourceId()
+ RENAME_LOG_RECORD_SEPARATOR
+ e.getValue())
.collect(toImmutableList());
writeOperationFile(
dst.getAuthority(),
dst.getBucketName(),
OPERATION_LOG_FILE_FORMAT,
CREATE_OBJECT_OPTIONS,
"rename",
CoopLockOperationType.RENAME,
operationId,
operationInstant,
logRecords);
// Schedule lock expiration update
lockUpdateFuture =
scheduleLockUpdate(
operationId,
operationLockPath,
RenameOperation.class,
(o, i) -> o.setLockEpochSeconds(i.getEpochSecond()));
return lockUpdateFuture;
return scheduleLockUpdate(
operationId,
operationLockPath,
RenameOperation.class,
(o, i) -> o.setLockEpochSeconds(i.getEpochSecond()));
}

public void checkpointUpdateOperation(
FileInfo srcInfo, URI dst, String operationId, Instant operationInstant) throws IOException {
public void checkpointRenameOperation(
StorageResourceId src,
StorageResourceId dst,
String operationId,
Instant operationInstant,
boolean copySucceeded)
throws IOException {
writeOperationFile(
dst.getAuthority(),
dst.getBucketName(),
OPERATION_LOCK_FILE_FORMAT,
UPDATE_OBJECT_OPTIONS,
"rename",
CoopLockOperationType.RENAME,
operationId,
operationInstant,
ImmutableList.of(
GSON.toJson(
new RenameOperation()
.setLockEpochSeconds(Instant.now().getEpochSecond())
.setSrcResource(srcInfo.getPath().toString())
.setSrcResource(src.toString())
.setDstResource(dst.toString())
.setCopySucceeded(true))));
.setCopySucceeded(copySucceeded))));
}

private void renewLockOrExit(
@@ -179,9 +202,10 @@ private void renewLockOrExit(
for (int i = 0; i < 10; i++) {
try {
renewLock(operationId, operationLockPath, renewFn);
return;
} catch (IOException e) {
logger.atWarning().withCause(e).log(
"Failed to renew '%s' lock for %s operation, retry #%d",
"Failed to renew '%s' lock for %s operation, attempt #%d",
operationLockPath, operationId, i + 1);
}
sleepUninterruptibly(100, TimeUnit.MILLISECONDS);
@@ -194,7 +218,8 @@ private void renewLockOrExit(
private void renewLock(
String operationId, URI operationLockPath, Function<String, String> renewFn)
throws IOException {
StorageResourceId lockId = StorageResourceId.fromObjectName(operationLockPath.toString());
StorageResourceId lockId =
pathCodec.validatePathAndGetId(operationLockPath, /* allowEmptyObjectNames =*/ false);
GoogleCloudStorageItemInfo lockInfo = gcs.getItemInfo(lockId);
checkState(lockInfo.exists(), "lock file for %s operation should exist", operationId);

@@ -208,30 +233,23 @@ private void renewLock(
CreateObjectOptions updateOptions =
new CreateObjectOptions(
/* overwriteExisting= */ true, DEFAULT_CONTENT_TYPE, EMPTY_METADATA);
StorageResourceId operationLockPathResourceId =
StorageResourceId lockIdWithGeneration =
new StorageResourceId(
operationLockPath.getAuthority(),
operationLockPath.getPath(),
lockInfo.getContentGeneration());
writeOperation(operationLockPathResourceId, updateOptions, ImmutableList.of(lock));
lockId.getBucketName(), lockId.getObjectName(), lockInfo.getContentGeneration());
writeOperation(lockIdWithGeneration, updateOptions, ImmutableList.of(lock));
}

private URI writeOperationFile(
String bucket,
String fileNameFormat,
CreateObjectOptions createObjectOptions,
String operation,
CoopLockOperationType operationType,
String operationId,
Instant operationInstant,
List<String> records)
throws IOException {
checkArgument(
VALID_OPERATIONS.contains(operation),
"operation must be one of $s, but was '%s'",
VALID_OPERATIONS,
operation);
String date = LOCK_FILE_DATE_TIME_FORMAT.format(operationInstant);
String file = String.format(LOCK_DIRECTORY + fileNameFormat, date, operation, operationId);
String file = String.format(LOCK_DIRECTORY + fileNameFormat, date, operationType, operationId);
URI path = pathCodec.getPath(bucket, file, /* allowEmptyObjectName= */ false);
StorageResourceId resourceId =
pathCodec.validatePathAndGetId(path, /* allowEmptyObjectName= */ false);
@@ -250,8 +268,6 @@ private void writeOperation(
}
}

private ScheduledExecutorService scheduledThreadPool = Executors.newScheduledThreadPool(1);

public <T> Future<?> scheduleLockUpdate(
String operationId, URI operationLockPath, Class<T> clazz, BiConsumer<T, Instant> renewFn) {
return scheduledThreadPool.scheduleAtFixedRate(
Original file line number Diff line number Diff line change
@@ -0,0 +1,110 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.DELETE;
import static com.google.common.base.Preconditions.checkArgument;

import com.google.cloud.hadoop.gcsio.FileInfo;
import com.google.cloud.hadoop.gcsio.ForwardingGoogleCloudStorage;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorage;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorageImpl;
import com.google.cloud.hadoop.gcsio.PathCodec;
import com.google.cloud.hadoop.gcsio.StorageResourceId;
import com.google.common.base.MoreObjects;
import java.io.IOException;
import java.net.URI;
import java.time.Instant;
import java.util.List;
import java.util.UUID;
import java.util.concurrent.Future;

public class CoopLockOperationDelete {

private final String operationId = UUID.randomUUID().toString();
private final Instant operationInstant = Instant.now();

private final StorageResourceId resourceId;

private final CoopLockRecordsDao coopLockRecordsDao;
private final CoopLockOperationDao coopLockOperationDao;

private Future<?> lockUpdateFuture;

private CoopLockOperationDelete(
GoogleCloudStorageImpl gcs, PathCodec pathCodec, StorageResourceId resourceId) {
this.resourceId = resourceId;
this.coopLockRecordsDao = new CoopLockRecordsDao(gcs);
this.coopLockOperationDao = new CoopLockOperationDao(gcs, pathCodec);
}

public static CoopLockOperationDelete create(
GoogleCloudStorage gcs, PathCodec pathCodec, URI path) {
while (gcs instanceof ForwardingGoogleCloudStorage) {
gcs = ((ForwardingGoogleCloudStorage) gcs).getDelegate();
}
checkArgument(
gcs instanceof GoogleCloudStorageImpl,
"gcs should be instance of %s, but was %s",
GoogleCloudStorageImpl.class,
gcs.getClass());
return new CoopLockOperationDelete(
(GoogleCloudStorageImpl) gcs,
pathCodec,
pathCodec.validatePathAndGetId(path, /* allowEmptyObjectName= */ true));
}

public void lock() {
try {
coopLockRecordsDao.lockPaths(operationId, operationInstant, DELETE, resourceId);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to acquire lock for %s operation", this), e);
}
}

public void persistAndScheduleRenewal(
List<FileInfo> itemsToDelete, List<FileInfo> bucketsToDelete) {
try {
lockUpdateFuture =
coopLockOperationDao.persistDeleteOperation(
operationId, operationInstant, resourceId, itemsToDelete, bucketsToDelete);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to persist %s operation", this), e);
}
}

public void unlock() {
try {
coopLockRecordsDao.unlockPaths(operationId, resourceId);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to release lock for %s operation", this), e);
}
}

public void cancelRenewal() {
lockUpdateFuture.cancel(/* mayInterruptIfRunning= */ true);
}

@Override
public String toString() {
return MoreObjects.toStringHelper(this)
.add("operationId", operationId)
.add("operationInstant", operationInstant)
.add("resourceId", resourceId)
.toString();
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,133 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.RENAME;
import static com.google.common.base.Preconditions.checkArgument;

import com.google.cloud.hadoop.gcsio.FileInfo;
import com.google.cloud.hadoop.gcsio.ForwardingGoogleCloudStorage;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorage;
import com.google.cloud.hadoop.gcsio.GoogleCloudStorageImpl;
import com.google.cloud.hadoop.gcsio.PathCodec;
import com.google.cloud.hadoop.gcsio.StorageResourceId;
import com.google.common.base.MoreObjects;
import java.io.IOException;
import java.net.URI;
import java.time.Instant;
import java.util.Map;
import java.util.UUID;
import java.util.concurrent.Future;

public class CoopLockOperationRename {

private final String operationId = UUID.randomUUID().toString();
private final Instant operationInstant = Instant.now();

private final StorageResourceId srcResourceId;
private final StorageResourceId dstResourceId;

private final CoopLockRecordsDao coopLockRecordsDao;
private final CoopLockOperationDao coopLockOperationDao;

private Future<?> lockUpdateFuture;

private CoopLockOperationRename(
GoogleCloudStorageImpl gcs,
PathCodec pathCodec,
StorageResourceId srcResourceId,
StorageResourceId dstResourceId) {
this.srcResourceId = srcResourceId;
this.dstResourceId = dstResourceId;
this.coopLockRecordsDao = new CoopLockRecordsDao(gcs);
this.coopLockOperationDao = new CoopLockOperationDao(gcs, pathCodec);
}

public static CoopLockOperationRename create(
GoogleCloudStorage gcs, PathCodec pathCodec, URI src, URI dst) {
while (gcs instanceof ForwardingGoogleCloudStorage) {
gcs = ((ForwardingGoogleCloudStorage) gcs).getDelegate();
}
checkArgument(
gcs instanceof GoogleCloudStorageImpl,
"gcs should be instance of %s, but was %s",
GoogleCloudStorageImpl.class,
gcs.getClass());
return new CoopLockOperationRename(
(GoogleCloudStorageImpl) gcs,
pathCodec,
pathCodec.validatePathAndGetId(src, /* allowEmptyObjectName= */ true),
pathCodec.validatePathAndGetId(dst, /* allowEmptyObjectName= */ true));
}

public void lock() {
try {
coopLockRecordsDao.lockPaths(
operationId, operationInstant, RENAME, srcResourceId, dstResourceId);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to acquire lock %s operation", this), e);
}
}

public void persistAndScheduleRenewal(
Map<FileInfo, URI> srcToDstItemNames, Map<FileInfo, URI> srcToDstMarkerItemNames) {
try {
lockUpdateFuture =
coopLockOperationDao.persistRenameOperation(
operationId,
operationInstant,
srcResourceId,
dstResourceId,
srcToDstItemNames,
srcToDstMarkerItemNames);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to persist %s operation", this), e);
}
}

public void checkpoint() {
try {
coopLockOperationDao.checkpointRenameOperation(
srcResourceId, dstResourceId, operationId, operationInstant, /* copySucceeded= */ true);
} catch (IOException e) {
throw new RuntimeException(String.format("Failed to checkpoint %s operation", this), e);
}
}

public void unlock() {
try {
coopLockRecordsDao.unlockPaths(operationId, srcResourceId, dstResourceId);
} catch (IOException e) {
throw new RuntimeException(
String.format("Failed to release unlock for %s operation", this), e);
}
}

public void cancelRenewal() {
lockUpdateFuture.cancel(/* mayInterruptIfRunning= */ true);
}

@Override
public String toString() {
return MoreObjects.toStringHelper(this)
.add("operationId", operationId)
.add("operationInstant", operationInstant)
.add("srcResourceId", srcResourceId)
.add("dstResourceId", dstResourceId)
.toString();
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,23 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

/** Enum that represent cooperative locking operation type */
public enum CoopLockOperationType {
RENAME,
DELETE
}
Original file line number Diff line number Diff line change
@@ -1,3 +1,19 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import com.google.common.base.MoreObjects;
@@ -6,10 +22,22 @@

/** Class that represent cooperative locking operation */
public class CoopLockRecord {
private String clientId;
private String operationId;
private long operationEpochSeconds;
private CoopLockOperationType operationType;
private long lockEpochSeconds;
private Set<String> resources = new TreeSet<>();

public String getClientId() {
return clientId;
}

public CoopLockRecord setClientId(String clientId) {
this.clientId = clientId;
return this;
}

public String getOperationId() {
return operationId;
}
@@ -19,6 +47,24 @@ public CoopLockRecord setOperationId(String operationId) {
return this;
}

public long getOperationEpochSeconds() {
return operationEpochSeconds;
}

public CoopLockRecord setOperationEpochSeconds(long operationEpochSeconds) {
this.operationEpochSeconds = operationEpochSeconds;
return this;
}

public CoopLockOperationType getOperationType() {
return operationType;
}

public CoopLockRecord setOperationType(CoopLockOperationType operationType) {
this.operationType = operationType;
return this;
}

public long getLockEpochSeconds() {
return lockEpochSeconds;
}
@@ -40,7 +86,10 @@ public CoopLockRecord setResources(Set<String> resources) {
@Override
public String toString() {
return MoreObjects.toStringHelper(this)
.add("clientId", clientId)
.add("operationId", operationId)
.add("operationEpochSeconds", operationEpochSeconds)
.add("operationType", operationType)
.add("lockEpochSeconds", lockEpochSeconds)
.add("resources", resources)
.toString();
Original file line number Diff line number Diff line change
@@ -1,3 +1,19 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import static com.google.common.base.Preconditions.checkState;
@@ -8,8 +24,13 @@
import java.util.TreeSet;

public class CoopLockRecords {
/** Supported version of operation locks */
public static final long FORMAT_VERSION = 1;
/**
* Supported version of operation locks persistent objects format.
*
* <p>When making any changes to cooperative locking persistent objects format (adding, renaming
* or removing fields), then you need to increase this version number to prevent corruption.
*/
public static final long FORMAT_VERSION = 2;

private long formatVersion = -1;
private Set<CoopLockRecord> locks =
Original file line number Diff line number Diff line change
@@ -1,5 +1,5 @@
/*
* Copyright 2019 Google Inc. All Rights Reserved.
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
@@ -35,6 +35,9 @@
import com.google.common.flogger.GoogleLogger;
import com.google.gson.Gson;
import java.io.IOException;
import java.net.InetAddress;
import java.net.UnknownHostException;
import java.time.Duration;
import java.time.Instant;
import java.util.Arrays;
import java.util.HashMap;
@@ -43,21 +46,22 @@
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import java.util.function.Function;
import java.util.stream.Collectors;

public class CoopLockRecordsDao {

private static final GoogleLogger logger = GoogleLogger.forEnclosingClass();

public static final String LOCK_DIRECTORY = "_lock/";

private static final Gson GSON = new Gson();

public static final String LOCK_FILE = "all.lock";
private static final String LOCK_FILE = "all.lock";
public static final String LOCK_PATH = LOCK_DIRECTORY + LOCK_FILE;

private static final String LOCK_METADATA_KEY = "lock";
private static final int MAX_LOCKS_COUNT = 20;
private static final int RETRY_INTERVAL_MILLIS = 2_000;

private static final Gson GSON = new Gson();

private final GoogleCloudStorageImpl gcs;

@@ -77,31 +81,56 @@ public Set<CoopLockRecord> getLockedOperations(String bucketName) throws IOExcep
? new HashSet<>()
: getLockRecords(lockInfo).getLocks();
logger.atFine().log(
"[%dms] lockPaths(%s): %s", System.currentTimeMillis() - startMs, bucketName, operations);
"[%dms] getLockedOperations(%s): %s",
System.currentTimeMillis() - startMs, bucketName, operations);
return operations;
}

public void lockOperation(String bucketName, String operationId, long lockEpochSeconds)
public void relockOperation(
String bucketName, String operationId, String clientId, long lockEpochSeconds)
throws IOException {
long startMs = System.currentTimeMillis();
logger.atFine().log("lockOperation(%s, %d)", operationId, lockEpochSeconds);
modifyLock(this::updateLockEpochSeconds, bucketName, operationId, lockEpochSeconds);
logger.atFine().log("lockOperation(%s, %d)", operationId, clientId);
modifyLock(
records -> reacquireOperationLock(records, operationId, clientId, lockEpochSeconds),
bucketName,
operationId);
logger.atFine().log(
"[%dms] lockOperation(%s, %s)",
System.currentTimeMillis() - startMs, operationId, lockEpochSeconds);
System.currentTimeMillis() - startMs, operationId, clientId);
}

public void lockPaths(String operationId, StorageResourceId... resources) throws IOException {
public void lockPaths(
String operationId,
Instant operationInstant,
CoopLockOperationType operationType,
StorageResourceId... resources)
throws IOException {
long startMs = System.currentTimeMillis();
logger.atFine().log("lockPaths(%s, %s)", operationId, lazy(() -> Arrays.toString(resources)));
Set<String> objects = validateResources(resources);
String bucketName = resources[0].getBucketName();
modifyLock(this::addLockRecords, bucketName, operationId, objects);
modifyLock(
records -> addLockRecords(records, operationId, operationInstant, operationType, objects),
bucketName,
operationId);
logger.atFine().log(
"[%dms] lockPaths(%s, %s)",
System.currentTimeMillis() - startMs, operationId, lazy(() -> Arrays.toString(resources)));
}

public void unlockPaths(String operationId, StorageResourceId... resources) throws IOException {
long startMs = System.currentTimeMillis();
logger.atFine().log("unlockPaths(%s, %s)", operationId, lazy(() -> Arrays.toString(resources)));
Set<String> objects = validateResources(resources);
String bucketName = resources[0].getBucketName();
modifyLock(
records -> removeLockRecords(records, operationId, objects), bucketName, operationId);
logger.atFine().log(
"[%dms] unlockPaths(%s, %s)",
System.currentTimeMillis() - startMs, operationId, lazy(() -> Arrays.toString(resources)));
}

private Set<String> validateResources(StorageResourceId[] resources) {
checkNotNull(resources, "resources should not be null");
checkArgument(resources.length > 0, "resources should not be empty");
@@ -113,22 +142,8 @@ private Set<String> validateResources(StorageResourceId[] resources) {
return Arrays.stream(resources).map(StorageResourceId::getObjectName).collect(toImmutableSet());
}

public void unlockPaths(String operationId, StorageResourceId... resources) throws IOException {
long startMs = System.currentTimeMillis();
logger.atFine().log("unlockPaths(%s, %s)", operationId, lazy(() -> Arrays.toString(resources)));
Set<String> objects = validateResources(resources);
String bucketName = resources[0].getBucketName();
modifyLock(this::removeLockRecords, bucketName, operationId, objects);
logger.atFine().log(
"[%dms] unlockPaths(%s, %s)",
System.currentTimeMillis() - startMs, operationId, lazy(() -> Arrays.toString(resources)));
}

private <T> void modifyLock(
LockRecordsModificationFunction<Boolean, CoopLockRecords, String, T> modificationFn,
String bucketName,
String operationId,
T modificationFnParam)
private void modifyLock(
Function<CoopLockRecords, Boolean> modificationFn, String bucketName, String operationId)
throws IOException {
long startMs = System.currentTimeMillis();
StorageResourceId lockId = getLockId(bucketName);
@@ -137,15 +152,15 @@ private <T> void modifyLock(
new ExponentialBackOff.Builder()
.setInitialIntervalMillis(100)
.setMultiplier(1.2)
.setMaxIntervalMillis(30_000)
.setMaxIntervalMillis((int) Duration.ofSeconds(MAX_LOCKS_COUNT).toMillis())
.setMaxElapsedTimeMillis(Integer.MAX_VALUE)
.build();

do {
try {
GoogleCloudStorageItemInfo lockInfo = gcs.getItemInfo(lockId);
if (!lockInfo.exists()) {
gcs.createEmptyObject(lockId, new CreateObjectOptions(false));
gcs.createEmptyObject(lockId, new CreateObjectOptions(/* overwriteExisting= */ false));
lockInfo = gcs.getItemInfo(lockId);
}
CoopLockRecords lockRecords =
@@ -154,25 +169,25 @@ private <T> void modifyLock(
? new CoopLockRecords().setFormatVersion(CoopLockRecords.FORMAT_VERSION)
: getLockRecords(lockInfo);

if (!modificationFn.apply(lockRecords, operationId, modificationFnParam)) {
if (!modificationFn.apply(lockRecords)) {
logger.atInfo().atMostEvery(5, SECONDS).log(
"Failed to update %s entries in %s file: resources could be locked. Re-trying.",
modificationFnParam, lockRecords.getLocks().size(), lockId);
sleepUninterruptibly(backOff.nextBackOffMillis(), MILLISECONDS);
"Failed to update %s entries in %s file: resources could be locked, retrying.",
lockRecords.getLocks().size(), lockId);
sleepUninterruptibly(RETRY_INTERVAL_MILLIS, MILLISECONDS);
continue;
}

// Unlocked all objects - delete lock object
// If unlocked all objects - delete lock object
if (lockRecords.getLocks().isEmpty()) {
gcs.deleteObject(lockInfo.getResourceId(), lockInfo.getMetaGeneration());
break;
}

if (lockRecords.getLocks().size() > MAX_LOCKS_COUNT) {
logger.atInfo().atMostEvery(5, SECONDS).log(
"Skipping lock entries update in %s file: too many (%d) locked resources. Re-trying.",
"Skipping lock entries update in %s file: too many (%d) locked resources, retrying.",
lockRecords.getLocks().size(), lockId);
sleepUninterruptibly(backOff.nextBackOffMillis(), MILLISECONDS);
sleepUninterruptibly(RETRY_INTERVAL_MILLIS, MILLISECONDS);
continue;
}

@@ -183,29 +198,27 @@ private <T> void modifyLock(
gcs.updateMetadata(lockInfo, metadata);

logger.atFine().log(
"Updated lock file in %dms for %s operation with %s parameter",
System.currentTimeMillis() - startMs, operationId, lazy(modificationFnParam::toString));
"Updated lock file in %dms for %s operation",
System.currentTimeMillis() - startMs, operationId);
break;
} catch (IOException e) {
// continue after sleep if update failed due to file generation mismatch or other
// IOException
if (e.getMessage().contains("conditionNotMet")) {
logger.atInfo().atMostEvery(5, SECONDS).log(
"Failed to update entries (conditionNotMet) in %s file for operation %s. Re-trying.",
"Failed to update entries (conditionNotMet) in %s file for operation %s, retrying.",
lockId, operationId);
} else {
logger.atWarning().withCause(e).log(
"Failed to modify lock for %s operation with %s parameter, retrying.",
operationId, modificationFnParam);
"Failed to modify lock for %s operation, retrying.", operationId);
}
sleepUninterruptibly(backOff.nextBackOffMillis(), MILLISECONDS);
}
} while (true);
}

private StorageResourceId getLockId(String bucketName) {
String lockObject = "gs://" + bucketName + "/" + LOCK_PATH;
return StorageResourceId.fromObjectName(lockObject);
return new StorageResourceId(bucketName, LOCK_PATH);
}

private CoopLockRecords getLockRecords(GoogleCloudStorageItemInfo lockInfo) {
@@ -219,51 +232,63 @@ private CoopLockRecords getLockRecords(GoogleCloudStorageItemInfo lockInfo) {
return lockRecords;
}

private boolean updateLockEpochSeconds(
CoopLockRecords lockRecords, String operationId, long lockEpochSeconds) {
private boolean reacquireOperationLock(
CoopLockRecords lockRecords, String operationId, String clientId, long lockEpochSeconds) {
Optional<CoopLockRecord> operationOptional =
lockRecords.getLocks().stream()
.filter(o -> o.getOperationId().equals(operationId))
.findAny();
checkState(operationOptional.isPresent(), "operation %s not found", operationId);
CoopLockRecord operation = operationOptional.get();
checkState(
lockEpochSeconds == operation.getLockEpochSeconds(),
"operation %s should have %s lock epoch, but was %s",
clientId.equals(operation.getClientId()),
"operation %s should be locked by %s client, but was %s",
operationId,
clientId,
operation.getClientId());
checkState(
lockEpochSeconds == operation.getLockEpochSeconds(),
"operation %s should be locked at %s epoch seconds but was at %s",
lockEpochSeconds,
operation.getLockEpochSeconds());
operation.setLockEpochSeconds(Instant.now().getEpochSecond());
return true;
}

private boolean addLockRecords(
CoopLockRecords lockRecords, String operationId, Set<String> resourcesToAdd) {
CoopLockRecords lockRecords,
String operationId,
Instant operationInstant,
CoopLockOperationType operationType,
Set<String> resourcesToAdd) {
// TODO: optimize to match more efficiently
if (lockRecords.getLocks().stream()
.flatMap(operation -> operation.getResources().stream())
.anyMatch(
resource -> {
for (String resourceToAdd : resourcesToAdd) {
if (resourceToAdd.equals(resource)
|| isChildObject(resource, resourceToAdd)
|| isChildObject(resourceToAdd, resource)) {
return true;
}
}
return false;
})) {
boolean atLestOneResourceAlreadyLocked =
lockRecords.getLocks().stream()
.flatMap(operation -> operation.getResources().stream())
.anyMatch(
lockedResource -> {
for (String resourceToAdd : resourcesToAdd) {
if (resourceToAdd.equals(lockedResource)
|| isChildObject(lockedResource, resourceToAdd)
|| isChildObject(resourceToAdd, lockedResource)) {
return true;
}
}
return false;
});
if (atLestOneResourceAlreadyLocked) {
return false;
}

long lockEpochSeconds = Instant.now().getEpochSecond();
lockRecords
.getLocks()
.add(
new CoopLockRecord()
.setOperationId(operationId)
.setResources(resourcesToAdd)
.setLockEpochSeconds(lockEpochSeconds));
CoopLockRecord record =
new CoopLockRecord()
.setClientId(newClientId(operationId))
.setOperationId(operationId)
.setOperationEpochSeconds(operationInstant.getEpochSecond())
.setLockEpochSeconds(Instant.now().getEpochSecond())
.setOperationType(operationType)
.setResources(resourcesToAdd);
lockRecords.getLocks().add(record);

return true;
}
@@ -274,17 +299,16 @@ private boolean isChildObject(String parent, String child) {

private boolean removeLockRecords(
CoopLockRecords lockRecords, String operationId, Set<String> resourcesToRemove) {
List<CoopLockRecord> operationLocksToRemoveRecord =
List<CoopLockRecord> recordsToRemove =
lockRecords.getLocks().stream()
.filter(o -> o.getResources().stream().anyMatch(resourcesToRemove::contains))
.collect(Collectors.toList());
checkState(
operationLocksToRemoveRecord.size() == 1
&& operationLocksToRemoveRecord.get(0).getOperationId().equals(operationId),
recordsToRemove.size() == 1 && recordsToRemove.get(0).getOperationId().equals(operationId),
"All resources %s should belong to %s operation, but was %s",
resourcesToRemove.size(),
operationLocksToRemoveRecord.size());
CoopLockRecord operationToRemove = operationLocksToRemoveRecord.get(0);
recordsToRemove.size());
CoopLockRecord operationToRemove = recordsToRemove.get(0);
checkState(
operationToRemove.getResources().equals(resourcesToRemove),
"All of %s resources should be locked by operation, but was locked only %s resources",
@@ -297,8 +321,15 @@ private boolean removeLockRecords(
return true;
}

@FunctionalInterface
private interface LockRecordsModificationFunction<T, T1, T2, T3> {
T apply(T1 p1, T2 p2, T3 p3);
private static String newClientId(String operationId) {
InetAddress localHost;
try {
localHost = InetAddress.getLocalHost();
} catch (UnknownHostException e) {
throw new RuntimeException(
String.format("Failed to get clientId for %s operation", operationId), e);
}
String epochMillis = String.valueOf(Instant.now().toEpochMilli());
return localHost.getCanonicalHostName() + "-" + epochMillis.substring(epochMillis.length() - 6);
}
}
Original file line number Diff line number Diff line change
@@ -1,3 +1,19 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import com.google.common.base.MoreObjects;
Original file line number Diff line number Diff line change
@@ -1,3 +1,19 @@
/*
* Copyright 2019 Google LLC. All Rights Reserved.
*
* Licensed under the Apache License, Version 2.0 (the "License");
* you may not use this file except in compliance with the License.
* You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

package com.google.cloud.hadoop.gcsio.cooplock;

import com.google.common.base.MoreObjects;
Original file line number Diff line number Diff line change
@@ -19,6 +19,8 @@
import static com.google.cloud.hadoop.gcsio.TrackingHttpRequestInitializer.deleteRequestString;
import static com.google.cloud.hadoop.gcsio.TrackingHttpRequestInitializer.updateMetadataRequestString;
import static com.google.cloud.hadoop.gcsio.TrackingHttpRequestInitializer.uploadRequestString;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.DELETE;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockOperationType.RENAME;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockRecordsDao.LOCK_DIRECTORY;
import static com.google.cloud.hadoop.gcsio.cooplock.CoopLockRecordsDao.LOCK_PATH;
import static com.google.common.base.Preconditions.checkNotNull;
@@ -50,7 +52,7 @@ public class CoopLockIntegrationTest {

private static final Gson GSON = new Gson();

private static final String OPERATION_FILENAME_PATTERN =
private static final String OPERATION_FILENAME_PATTERN_FORMAT =
"[0-9]{8}T[0-9]{6}\\.[0-9]{3}Z_%s_[a-z0-9\\-]+";

private static GoogleCloudStorageOptions gcsOptions;
@@ -132,10 +134,9 @@ public void moveDirectory() throws Exception {
.collect(toList());

assertThat(lockFiles).hasSize(2);
URI lockFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.lock").get();
URI logFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "rename") + "\\.log").get();
String fileNamePattern = String.format(OPERATION_FILENAME_PATTERN_FORMAT, RENAME);
URI lockFileUri = matchFile(lockFiles, fileNamePattern + "\\.lock").get();
URI logFileUri = matchFile(lockFiles, fileNamePattern + "\\.log").get();

String lockContent = gcsfsIHelper.readTextFile(bucketName, lockFileUri.getPath());
assertThat(GSON.fromJson(lockContent, RenameOperation.class).setLockEpochSeconds(0))
@@ -182,11 +183,14 @@ public void deleteDirectory() throws Exception {
.collect(toList());

assertThat(lockFiles).hasSize(2);
URI lockFileUri =
matchFile(lockFiles, String.format(OPERATION_FILENAME_PATTERN, "delete") + "\\.lock").get();
String fileNamePattern = String.format(OPERATION_FILENAME_PATTERN_FORMAT, DELETE);
URI lockFileUri = matchFile(lockFiles, fileNamePattern + "\\.lock").get();
URI logFileUri = matchFile(lockFiles, fileNamePattern + "\\.log").get();
String lockContent = gcsfsIHelper.readTextFile(bucketName, lockFileUri.getPath());
assertThat(GSON.fromJson(lockContent, DeleteOperation.class).setLockEpochSeconds(0))
.isEqualTo(new DeleteOperation().setLockEpochSeconds(0).setResource(dirUri.toString()));
assertThat(gcsfsIHelper.readTextFile(bucketName, logFileUri.getPath()))
.isEqualTo(dirUri.resolve(fileName) + "\n" + dirUri + "\n");
}

private Optional<URI> matchFile(List<URI> files, String pattern) {

0 comments on commit 34bca3a

Please sign in to comment.