Skip to content

Commit

Permalink
Merge pull request #52047 from ClickHouse/backport/23.6/51954
Browse files Browse the repository at this point in the history
Backport #51954 to 23.6: Check refcount in `RemoveManyObjectStorageOperation::finalize` instead of `execute`
  • Loading branch information
alexey-milovidov committed Jul 15, 2023
2 parents ab0fdf2 + 4fdf8a3 commit d04e4a7
Show file tree
Hide file tree
Showing 10 changed files with 184 additions and 44 deletions.
73 changes: 42 additions & 31 deletions src/Disks/ObjectStorages/DiskObjectStorageTransaction.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -6,6 +6,8 @@
#include <Common/logger_useful.h>
#include <Common/Exception.h>

#include <Disks/ObjectStorages/MetadataStorageFromDisk.h>

namespace DB
{

Expand Down Expand Up @@ -63,11 +65,18 @@ struct PureMetadataObjectStorageOperation final : public IDiskObjectStorageOpera
std::string getInfoForLog() const override { return fmt::format("PureMetadataObjectStorageOperation"); }
};


struct ObjectsToRemove
{
StoredObjects objects;
UnlinkMetadataFileOperationOutcomePtr unlink_outcome;
};

struct RemoveObjectStorageOperation final : public IDiskObjectStorageOperation
{
std::string path;
bool delete_metadata_only;
StoredObjects objects_to_remove;
ObjectsToRemove objects_to_remove;
bool if_exists;
bool remove_from_cache = false;

Expand Down Expand Up @@ -103,15 +112,12 @@ struct RemoveObjectStorageOperation final : public IDiskObjectStorageOperation

try
{
uint32_t hardlink_count = metadata_storage.getHardlinkCount(path);
auto objects = metadata_storage.getStorageObjects(path);

tx->unlinkMetadata(path);
auto unlink_outcome = tx->unlinkMetadata(path);

if (hardlink_count == 0)
{
objects_to_remove = std::move(objects);
}
if (unlink_outcome)
objects_to_remove = ObjectsToRemove{std::move(objects), std::move(unlink_outcome)};
}
catch (const Exception & e)
{
Expand Down Expand Up @@ -140,8 +146,11 @@ struct RemoveObjectStorageOperation final : public IDiskObjectStorageOperation
/// due to network error or similar. And when it will retry an operation it may receive
/// a 404 HTTP code. We don't want to threat this code as a real error for deletion process
/// (e.g. throwing some exceptions) and thus we just use method `removeObjectsIfExists`
if (!delete_metadata_only && !objects_to_remove.empty())
object_storage.removeObjectsIfExist(objects_to_remove);
if (!delete_metadata_only && !objects_to_remove.objects.empty()
&& objects_to_remove.unlink_outcome->num_hardlinks == 0)
{
object_storage.removeObjectsIfExist(objects_to_remove.objects);
}
}
};

Expand All @@ -150,7 +159,9 @@ struct RemoveManyObjectStorageOperation final : public IDiskObjectStorageOperati
RemoveBatchRequest remove_paths;
bool keep_all_batch_data;
NameSet file_names_remove_metadata_only;
StoredObjects objects_to_remove;

std::vector<ObjectsToRemove> objects_to_remove;

bool remove_from_cache = false;

RemoveManyObjectStorageOperation(
Expand All @@ -174,7 +185,6 @@ struct RemoveManyObjectStorageOperation final : public IDiskObjectStorageOperati
{
for (const auto & [path, if_exists] : remove_paths)
{

if (!metadata_storage.exists(path))
{
if (if_exists)
Expand All @@ -188,14 +198,12 @@ struct RemoveManyObjectStorageOperation final : public IDiskObjectStorageOperati

try
{
uint32_t hardlink_count = metadata_storage.getHardlinkCount(path);
auto objects = metadata_storage.getStorageObjects(path);

tx->unlinkMetadata(path);

/// File is really redundant
if (hardlink_count == 0 && !keep_all_batch_data && !file_names_remove_metadata_only.contains(fs::path(path).filename()))
std::move(objects.begin(), objects.end(), std::back_inserter(objects_to_remove));
auto unlink_outcome = tx->unlinkMetadata(path);
if (unlink_outcome && !keep_all_batch_data && !file_names_remove_metadata_only.contains(fs::path(path).filename()))
{
objects_to_remove.emplace_back(ObjectsToRemove{std::move(objects), std::move(unlink_outcome)});
}
}
catch (const Exception & e)
{
Expand All @@ -215,26 +223,31 @@ struct RemoveManyObjectStorageOperation final : public IDiskObjectStorageOperati

void undo() override
{

}

void finalize() override
{
StoredObjects remove_from_remote;
for (auto && [objects, unlink_outcome] : objects_to_remove)
{
if (unlink_outcome->num_hardlinks == 0)
std::move(objects.begin(), objects.end(), std::back_inserter(remove_from_remote));
}

/// Read comment inside RemoveObjectStorageOperation class
/// TL;DR Don't pay any attention to 404 status code
if (!objects_to_remove.empty())
object_storage.removeObjectsIfExist(objects_to_remove);
if (!remove_from_remote.empty())
object_storage.removeObjectsIfExist(remove_from_remote);
}
};


struct RemoveRecursiveObjectStorageOperation final : public IDiskObjectStorageOperation
{
std::string path;
std::unordered_map<std::string, StoredObjects> objects_to_remove;
std::unordered_map<std::string, ObjectsToRemove> objects_to_remove_by_path;
bool keep_all_batch_data;
NameSet file_names_remove_metadata_only;
StoredObjects objects_to_remove_from_cache;

RemoveRecursiveObjectStorageOperation(
IObjectStorage & object_storage_,
Expand All @@ -261,14 +274,11 @@ struct RemoveRecursiveObjectStorageOperation final : public IDiskObjectStorageOp
{
try
{
uint32_t hardlink_count = metadata_storage.getHardlinkCount(path_to_remove);
auto objects_paths = metadata_storage.getStorageObjects(path_to_remove);

tx->unlinkMetadata(path_to_remove);

if (hardlink_count == 0)
auto unlink_outcome = tx->unlinkMetadata(path_to_remove);
if (unlink_outcome)
{
objects_to_remove[path_to_remove] = std::move(objects_paths);
objects_to_remove_by_path[path_to_remove] = ObjectsToRemove{std::move(objects_paths), std::move(unlink_outcome)};
}
}
catch (const Exception & e)
Expand Down Expand Up @@ -318,11 +328,12 @@ struct RemoveRecursiveObjectStorageOperation final : public IDiskObjectStorageOp
if (!keep_all_batch_data)
{
StoredObjects remove_from_remote;
for (auto && [local_path, remote_paths] : objects_to_remove)
for (auto && [local_path, objects_to_remove] : objects_to_remove_by_path)
{
if (!file_names_remove_metadata_only.contains(fs::path(local_path).filename()))
{
std::move(remote_paths.begin(), remote_paths.end(), std::back_inserter(remove_from_remote));
if (objects_to_remove.unlink_outcome->num_hardlinks == 0)
std::move(objects_to_remove.objects.begin(), objects_to_remove.objects.end(), std::back_inserter(remove_from_remote));
}
}
/// Read comment inside RemoveObjectStorageOperation class
Expand Down
5 changes: 4 additions & 1 deletion src/Disks/ObjectStorages/IMetadataStorage.h
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,8 @@ namespace ErrorCodes
}

class IMetadataStorage;
struct UnlinkMetadataFileOperationOutcome;
using UnlinkMetadataFileOperationOutcomePtr = std::shared_ptr<UnlinkMetadataFileOperationOutcome>;

/// Tries to provide some "transactions" interface, which allow
/// to execute (commit) operations simultaneously. We don't provide
Expand Down Expand Up @@ -127,9 +129,10 @@ class IMetadataTransaction : private boost::noncopyable

/// Unlink metadata file and do something special if required
/// By default just remove file (unlink file).
virtual void unlinkMetadata(const std::string & path)
virtual UnlinkMetadataFileOperationOutcomePtr unlinkMetadata(const std::string & path)
{
unlinkFile(path);
return nullptr;
}

virtual ~IMetadataTransaction() = default;
Expand Down
7 changes: 5 additions & 2 deletions src/Disks/ObjectStorages/MetadataStorageFromDisk.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -340,9 +340,12 @@ void MetadataStorageFromDiskTransaction::addBlobToMetadata(const std::string & p
addOperation(std::make_unique<AddBlobOperation>(path, blob_name, metadata_storage.object_storage_root_path, size_in_bytes, *metadata_storage.disk, metadata_storage));
}

void MetadataStorageFromDiskTransaction::unlinkMetadata(const std::string & path)
UnlinkMetadataFileOperationOutcomePtr MetadataStorageFromDiskTransaction::unlinkMetadata(const std::string & path)
{
addOperation(std::make_unique<UnlinkMetadataFileOperation>(path, *metadata_storage.disk, metadata_storage));
auto operation = std::make_unique<UnlinkMetadataFileOperation>(path, *metadata_storage.getDisk(), metadata_storage);
auto result = operation->outcome;
addOperation(std::move(operation));
return result;
}

}
5 changes: 4 additions & 1 deletion src/Disks/ObjectStorages/MetadataStorageFromDisk.h
Original file line number Diff line number Diff line change
Expand Up @@ -11,6 +11,9 @@
namespace DB
{

struct UnlinkMetadataFileOperationOutcome;
using UnlinkMetadataFileOperationOutcomePtr = std::shared_ptr<UnlinkMetadataFileOperationOutcome>;

/// Store metadata on a separate disk
/// (used for object storages, like S3 and related).
class MetadataStorageFromDisk final : public IMetadataStorage
Expand Down Expand Up @@ -131,7 +134,7 @@ class MetadataStorageFromDiskTransaction final : public IMetadataTransaction

void replaceFile(const std::string & path_from, const std::string & path_to) override;

void unlinkMetadata(const std::string & path) override;
UnlinkMetadataFileOperationOutcomePtr unlinkMetadata(const std::string & path) override;


};
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -319,6 +319,8 @@ void UnlinkMetadataFileOperation::execute(std::unique_lock<SharedMutex> & metada
write_operation = std::make_unique<WriteFileOperation>(path, disk, metadata->serializeToString());
write_operation->execute(metadata_lock);
}
outcome->num_hardlinks = ref_count;

unlink_operation = std::make_unique<UnlinkFileOperation>(path, disk);
unlink_operation->execute(metadata_lock);
}
Expand All @@ -334,6 +336,9 @@ void UnlinkMetadataFileOperation::undo()

if (write_operation)
write_operation->undo();

/// Update outcome to reflect the fact that we have restored the file.
outcome->num_hardlinks++;
}

void SetReadonlyFileOperation::execute(std::unique_lock<SharedMutex> & metadata_lock)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -3,6 +3,8 @@
#include <Common/SharedMutex.h>
#include <Disks/ObjectStorages/IMetadataStorage.h>

#include <numeric>

namespace DB
{
class MetadataStorageFromDisk;
Expand Down Expand Up @@ -242,9 +244,19 @@ struct AddBlobOperation final : public IMetadataOperation
std::unique_ptr<WriteFileOperation> write_operation;
};

/// Return the result of operation to the caller.
/// It is used in `IDiskObjectStorageOperation::finalize` after metadata transaction executed to make decision on blob removal.
struct UnlinkMetadataFileOperationOutcome
{
UInt32 num_hardlinks = std::numeric_limits<UInt32>::max();
};

using UnlinkMetadataFileOperationOutcomePtr = std::shared_ptr<UnlinkMetadataFileOperationOutcome>;

struct UnlinkMetadataFileOperation final : public IMetadataOperation
{
const UnlinkMetadataFileOperationOutcomePtr outcome = std::make_shared<UnlinkMetadataFileOperationOutcome>();

UnlinkMetadataFileOperation(
const std::string & path_,
IDisk & disk_,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -135,9 +135,11 @@ void MetadataStorageFromPlainObjectStorageTransaction::addBlobToMetadata(
{
/// Noop, local metadata files is only one file, it is the metadata file itself.
}
void MetadataStorageFromPlainObjectStorageTransaction::unlinkMetadata(const std::string &)

UnlinkMetadataFileOperationOutcomePtr MetadataStorageFromPlainObjectStorageTransaction::unlinkMetadata(const std::string &)
{
/// Noop, no separate metadata.
/// No hardlinks, so will always remove file.
return std::make_shared<UnlinkMetadataFileOperationOutcome>(UnlinkMetadataFileOperationOutcome{0});
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -9,6 +9,9 @@
namespace DB
{

struct UnlinkMetadataFileOperationOutcome;
using UnlinkMetadataFileOperationOutcomePtr = std::shared_ptr<UnlinkMetadataFileOperationOutcome>;

/// Object storage is used as a filesystem, in a limited form:
/// - no directory concept, files only
/// - no stat/chmod/...
Expand Down Expand Up @@ -104,7 +107,7 @@ class MetadataStorageFromPlainObjectStorageTransaction final : public IMetadataT

void unlinkFile(const std::string & path) override;

void unlinkMetadata(const std::string & path) override;
UnlinkMetadataFileOperationOutcomePtr unlinkMetadata(const std::string & path) override;

void commit() override
{
Expand Down
Original file line number Diff line number Diff line change
@@ -1,12 +1,18 @@
<clickhouse>
<storage_configuration>
<disks>
<s3>
<s31>
<type>s3</type>
<endpoint>http://minio1:9001/root/data/</endpoint>
<access_key_id>minio</access_key_id>
<secret_access_key>minio123</secret_access_key>
</s3>
</s31>
<s32>
<type>s3</type>
<endpoint>http://minio1:9001/root/data2/</endpoint>
<access_key_id>minio</access_key_id>
<secret_access_key>minio123</secret_access_key>
</s32>
</disks>
<policies>
<two_disks>
Expand All @@ -15,10 +21,17 @@
<disk>default</disk>
</default>
<external>
<disk>s3</disk>
<disk>s31</disk>
</external>
</volumes>
</two_disks>
<one_disk>
<volumes>
<external>
<disk>s32</disk>
</external>
</volumes>
</one_disk>
</policies>
</storage_configuration>

Expand Down

0 comments on commit d04e4a7

Please sign in to comment.