mirror of
https://github.com/ClickHouse/ClickHouse.git
synced 2024-11-29 11:02:08 +00:00
Merge pull request #30658 from ClickHouse/std_mutex_alter_lock
Change `alter_lock` from `RWLock` to `std::timed_mutex`
This commit is contained in:
commit
2af950d4d5
@ -76,7 +76,8 @@ BlockIO InterpreterAlterQuery::executeToTable(const ASTAlterQuery & alter)
|
|||||||
StoragePtr table = DatabaseCatalog::instance().getTable(table_id, getContext());
|
StoragePtr table = DatabaseCatalog::instance().getTable(table_id, getContext());
|
||||||
if (table->isStaticStorage())
|
if (table->isStaticStorage())
|
||||||
throw Exception(ErrorCodes::TABLE_IS_READ_ONLY, "Table is read-only");
|
throw Exception(ErrorCodes::TABLE_IS_READ_ONLY, "Table is read-only");
|
||||||
auto alter_lock = table->lockForAlter(getContext()->getCurrentQueryId(), getContext()->getSettingsRef().lock_acquire_timeout);
|
auto table_lock = table->lockForShare(getContext()->getCurrentQueryId(), getContext()->getSettingsRef().lock_acquire_timeout);
|
||||||
|
auto alter_lock = table->lockForAlter(getContext()->getSettingsRef().lock_acquire_timeout);
|
||||||
auto metadata_snapshot = table->getInMemoryMetadataPtr();
|
auto metadata_snapshot = table->getInMemoryMetadataPtr();
|
||||||
|
|
||||||
/// Add default database to table identifiers that we can encounter in e.g. default expressions, mutation expression, etc.
|
/// Add default database to table identifiers that we can encounter in e.g. default expressions, mutation expression, etc.
|
||||||
|
@ -60,27 +60,31 @@ TableLockHolder IStorage::lockForShare(const String & query_id, const std::chron
|
|||||||
return result;
|
return result;
|
||||||
}
|
}
|
||||||
|
|
||||||
TableLockHolder IStorage::lockForAlter(const String & query_id, const std::chrono::milliseconds & acquire_timeout)
|
IStorage::AlterLockHolder IStorage::lockForAlter(const std::chrono::milliseconds & acquire_timeout)
|
||||||
{
|
{
|
||||||
TableLockHolder result = tryLockTimed(alter_lock, RWLockImpl::Write, query_id, acquire_timeout);
|
AlterLockHolder lock{alter_lock, std::defer_lock};
|
||||||
|
|
||||||
|
if (!lock.try_lock_for(acquire_timeout))
|
||||||
|
throw Exception(ErrorCodes::DEADLOCK_AVOIDED,
|
||||||
|
"Locking attempt for ALTER on \"{}\" has timed out! ({} ms) "
|
||||||
|
"Possible deadlock avoided. Client should retry.",
|
||||||
|
getStorageID().getFullTableName(), std::to_string(acquire_timeout.count()));
|
||||||
|
|
||||||
if (is_dropped)
|
if (is_dropped)
|
||||||
throw Exception("Table is dropped", ErrorCodes::TABLE_IS_DROPPED);
|
throw Exception("Table is dropped", ErrorCodes::TABLE_IS_DROPPED);
|
||||||
|
|
||||||
return result;
|
return lock;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
TableExclusiveLockHolder IStorage::lockExclusively(const String & query_id, const std::chrono::milliseconds & acquire_timeout)
|
TableExclusiveLockHolder IStorage::lockExclusively(const String & query_id, const std::chrono::milliseconds & acquire_timeout)
|
||||||
{
|
{
|
||||||
TableExclusiveLockHolder result;
|
TableExclusiveLockHolder result;
|
||||||
result.alter_lock = tryLockTimed(alter_lock, RWLockImpl::Write, query_id, acquire_timeout);
|
result.drop_lock = tryLockTimed(drop_lock, RWLockImpl::Write, query_id, acquire_timeout);
|
||||||
|
|
||||||
if (is_dropped)
|
if (is_dropped)
|
||||||
throw Exception("Table is dropped", ErrorCodes::TABLE_IS_DROPPED);
|
throw Exception("Table is dropped", ErrorCodes::TABLE_IS_DROPPED);
|
||||||
|
|
||||||
result.drop_lock = tryLockTimed(drop_lock, RWLockImpl::Write, query_id, acquire_timeout);
|
|
||||||
|
|
||||||
return result;
|
return result;
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -126,7 +130,7 @@ Pipe IStorage::alterPartition(
|
|||||||
throw Exception("Partition operations are not supported by storage " + getName(), ErrorCodes::NOT_IMPLEMENTED);
|
throw Exception("Partition operations are not supported by storage " + getName(), ErrorCodes::NOT_IMPLEMENTED);
|
||||||
}
|
}
|
||||||
|
|
||||||
void IStorage::alter(const AlterCommands & params, ContextPtr context, TableLockHolder &)
|
void IStorage::alter(const AlterCommands & params, ContextPtr context, AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
StorageInMemoryMetadata new_metadata = getInMemoryMetadata();
|
StorageInMemoryMetadata new_metadata = getInMemoryMetadata();
|
||||||
|
@ -238,7 +238,8 @@ public:
|
|||||||
|
|
||||||
/// Lock table for alter. This lock must be acuqired in ALTER queries to be
|
/// Lock table for alter. This lock must be acuqired in ALTER queries to be
|
||||||
/// sure, that we execute only one simultaneous alter. Doesn't affect share lock.
|
/// sure, that we execute only one simultaneous alter. Doesn't affect share lock.
|
||||||
TableLockHolder lockForAlter(const String & query_id, const std::chrono::milliseconds & acquire_timeout);
|
using AlterLockHolder = std::unique_lock<std::timed_mutex>;
|
||||||
|
AlterLockHolder lockForAlter(const std::chrono::milliseconds & acquire_timeout);
|
||||||
|
|
||||||
/// Lock table exclusively. This lock must be acquired if you want to be
|
/// Lock table exclusively. This lock must be acquired if you want to be
|
||||||
/// sure, that no other thread (SELECT, merge, ALTER, etc.) doing something
|
/// sure, that no other thread (SELECT, merge, ALTER, etc.) doing something
|
||||||
@ -417,7 +418,7 @@ public:
|
|||||||
/** ALTER tables in the form of column changes that do not affect the change
|
/** ALTER tables in the form of column changes that do not affect the change
|
||||||
* to Storage or its parameters. Executes under alter lock (lockForAlter).
|
* to Storage or its parameters. Executes under alter lock (lockForAlter).
|
||||||
*/
|
*/
|
||||||
virtual void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & alter_lock_holder);
|
virtual void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & alter_lock_holder);
|
||||||
|
|
||||||
/** Checks that alter commands can be applied to storage. For example, columns can be modified,
|
/** Checks that alter commands can be applied to storage. For example, columns can be modified,
|
||||||
* or primary key can be changes, etc.
|
* or primary key can be changes, etc.
|
||||||
@ -596,12 +597,9 @@ public:
|
|||||||
virtual bool dropTableImmediately() { return false; }
|
virtual bool dropTableImmediately() { return false; }
|
||||||
|
|
||||||
private:
|
private:
|
||||||
/// Lock required for alter queries (lockForAlter). Always taken for write
|
/// Lock required for alter queries (lockForAlter).
|
||||||
/// (actually can be replaced with std::mutex, but for consistency we use
|
/// Allows to execute only one simultaneous alter query.
|
||||||
/// RWLock). Allows to execute only one simultaneous alter query. Also it
|
mutable std::timed_mutex alter_lock;
|
||||||
/// should be taken by DROP-like queries, to be sure, that all alters are
|
|
||||||
/// finished.
|
|
||||||
mutable RWLock alter_lock = RWLockImpl::create();
|
|
||||||
|
|
||||||
/// Lock required for drop queries. Every thread that want to ensure, that
|
/// Lock required for drop queries. Every thread that want to ensure, that
|
||||||
/// table is not dropped have to table this lock for read (lockForShare).
|
/// table is not dropped have to table this lock for read (lockForShare).
|
||||||
|
@ -2211,7 +2211,7 @@ MergeTreeData::MutableDataPartPtr MergeTreeData::createPart(
|
|||||||
|
|
||||||
void MergeTreeData::changeSettings(
|
void MergeTreeData::changeSettings(
|
||||||
const ASTPtr & new_settings,
|
const ASTPtr & new_settings,
|
||||||
TableLockHolder & /* table_lock_holder */)
|
AlterLockHolder & /* table_lock_holder */)
|
||||||
{
|
{
|
||||||
if (new_settings)
|
if (new_settings)
|
||||||
{
|
{
|
||||||
|
@ -583,7 +583,7 @@ public:
|
|||||||
/// Change MergeTreeSettings
|
/// Change MergeTreeSettings
|
||||||
void changeSettings(
|
void changeSettings(
|
||||||
const ASTPtr & new_settings,
|
const ASTPtr & new_settings,
|
||||||
TableLockHolder & table_lock_holder);
|
AlterLockHolder & table_lock_holder);
|
||||||
|
|
||||||
/// Should be called if part data is suspected to be corrupted.
|
/// Should be called if part data is suspected to be corrupted.
|
||||||
void reportBrokenPart(const String & name) const
|
void reportBrokenPart(const String & name) const
|
||||||
|
@ -1066,7 +1066,7 @@ std::optional<UInt64> StorageBuffer::totalBytes(const Settings & /*settings*/) c
|
|||||||
return total_writes.bytes;
|
return total_writes.bytes;
|
||||||
}
|
}
|
||||||
|
|
||||||
void StorageBuffer::alter(const AlterCommands & params, ContextPtr local_context, TableLockHolder &)
|
void StorageBuffer::alter(const AlterCommands & params, ContextPtr local_context, AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
checkAlterIsPossible(params, local_context);
|
checkAlterIsPossible(params, local_context);
|
||||||
|
@ -107,7 +107,7 @@ public:
|
|||||||
void checkAlterIsPossible(const AlterCommands & commands, ContextPtr context) const override;
|
void checkAlterIsPossible(const AlterCommands & commands, ContextPtr context) const override;
|
||||||
|
|
||||||
/// The structure of the subordinate table is not checked and does not change.
|
/// The structure of the subordinate table is not checked and does not change.
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
std::optional<UInt64> totalRows(const Settings & settings) const override;
|
std::optional<UInt64> totalRows(const Settings & settings) const override;
|
||||||
std::optional<UInt64> totalBytes(const Settings & settings) const override;
|
std::optional<UInt64> totalBytes(const Settings & settings) const override;
|
||||||
|
@ -785,7 +785,7 @@ void StorageDistributed::checkAlterIsPossible(const AlterCommands & commands, Co
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
void StorageDistributed::alter(const AlterCommands & params, ContextPtr local_context, TableLockHolder &)
|
void StorageDistributed::alter(const AlterCommands & params, ContextPtr local_context, AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
|
|
||||||
|
@ -94,7 +94,7 @@ public:
|
|||||||
|
|
||||||
/// in the sub-tables, you need to manually add and delete columns
|
/// in the sub-tables, you need to manually add and delete columns
|
||||||
/// the structure of the sub-table is not checked
|
/// the structure of the sub-table is not checked
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
void startup() override;
|
void startup() override;
|
||||||
void shutdown() override;
|
void shutdown() override;
|
||||||
|
@ -274,7 +274,7 @@ bool StorageMaterializedView::optimize(
|
|||||||
void StorageMaterializedView::alter(
|
void StorageMaterializedView::alter(
|
||||||
const AlterCommands & params,
|
const AlterCommands & params,
|
||||||
ContextPtr local_context,
|
ContextPtr local_context,
|
||||||
TableLockHolder &)
|
AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
StorageInMemoryMetadata new_metadata = getInMemoryMetadata();
|
StorageInMemoryMetadata new_metadata = getInMemoryMetadata();
|
||||||
|
@ -50,7 +50,7 @@ public:
|
|||||||
const Names & deduplicate_by_columns,
|
const Names & deduplicate_by_columns,
|
||||||
ContextPtr context) override;
|
ContextPtr context) override;
|
||||||
|
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
void checkMutationIsPossible(const MutationCommands & commands, const Settings & settings) const override;
|
void checkMutationIsPossible(const MutationCommands & commands, const Settings & settings) const override;
|
||||||
|
|
||||||
|
@ -651,7 +651,7 @@ void StorageMerge::checkAlterIsPossible(const AlterCommands & commands, ContextP
|
|||||||
}
|
}
|
||||||
|
|
||||||
void StorageMerge::alter(
|
void StorageMerge::alter(
|
||||||
const AlterCommands & params, ContextPtr local_context, TableLockHolder &)
|
const AlterCommands & params, ContextPtr local_context, AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
|
|
||||||
|
@ -43,7 +43,7 @@ public:
|
|||||||
|
|
||||||
/// you need to add and remove columns in the sub-tables manually
|
/// you need to add and remove columns in the sub-tables manually
|
||||||
/// the structure of sub-tables is not checked
|
/// the structure of sub-tables is not checked
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
bool mayBenefitFromIndexForIn(
|
bool mayBenefitFromIndexForIn(
|
||||||
const ASTPtr & left_in_operand, ContextPtr query_context, const StorageMetadataPtr & metadata_snapshot) const override;
|
const ASTPtr & left_in_operand, ContextPtr query_context, const StorageMetadataPtr & metadata_snapshot) const override;
|
||||||
|
@ -271,7 +271,7 @@ void StorageMergeTree::truncate(const ASTPtr &, const StorageMetadataPtr &, Cont
|
|||||||
void StorageMergeTree::alter(
|
void StorageMergeTree::alter(
|
||||||
const AlterCommands & commands,
|
const AlterCommands & commands,
|
||||||
ContextPtr local_context,
|
ContextPtr local_context,
|
||||||
TableLockHolder & table_lock_holder)
|
AlterLockHolder & table_lock_holder)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
auto old_storage_settings = getSettings();
|
auto old_storage_settings = getSettings();
|
||||||
|
@ -86,7 +86,7 @@ public:
|
|||||||
void drop() override;
|
void drop() override;
|
||||||
void truncate(const ASTPtr &, const StorageMetadataPtr &, ContextPtr, TableExclusiveLockHolder &) override;
|
void truncate(const ASTPtr &, const StorageMetadataPtr &, ContextPtr, TableExclusiveLockHolder &) override;
|
||||||
|
|
||||||
void alter(const AlterCommands & commands, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & commands, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
void checkTableCanBeDropped() const override;
|
void checkTableCanBeDropped() const override;
|
||||||
|
|
||||||
|
@ -64,7 +64,7 @@ void StorageNull::checkAlterIsPossible(const AlterCommands & commands, ContextPt
|
|||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
void StorageNull::alter(const AlterCommands & params, ContextPtr context, TableLockHolder &)
|
void StorageNull::alter(const AlterCommands & params, ContextPtr context, AlterLockHolder &)
|
||||||
{
|
{
|
||||||
auto table_id = getStorageID();
|
auto table_id = getStorageID();
|
||||||
|
|
||||||
|
@ -43,7 +43,7 @@ public:
|
|||||||
|
|
||||||
void checkAlterIsPossible(const AlterCommands & commands, ContextPtr context) const override;
|
void checkAlterIsPossible(const AlterCommands & commands, ContextPtr context) const override;
|
||||||
|
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
std::optional<UInt64> totalRows(const Settings &) const override
|
std::optional<UInt64> totalRows(const Settings &) const override
|
||||||
{
|
{
|
||||||
|
@ -91,7 +91,7 @@ public:
|
|||||||
IStorage::renameInMemory(new_table_id);
|
IStorage::renameInMemory(new_table_id);
|
||||||
}
|
}
|
||||||
|
|
||||||
void alter(const AlterCommands & params, ContextPtr context, TableLockHolder & alter_lock_holder) override
|
void alter(const AlterCommands & params, ContextPtr context, AlterLockHolder & alter_lock_holder) override
|
||||||
{
|
{
|
||||||
getNested()->alter(params, context, alter_lock_holder);
|
getNested()->alter(params, context, alter_lock_holder);
|
||||||
IStorage::setInMemoryMetadata(getNested()->getInMemoryMetadata());
|
IStorage::setInMemoryMetadata(getNested()->getInMemoryMetadata());
|
||||||
|
@ -4355,7 +4355,8 @@ bool StorageReplicatedMergeTree::executeMetadataAlter(const StorageReplicatedMer
|
|||||||
zookeeper->multi(requests);
|
zookeeper->multi(requests);
|
||||||
|
|
||||||
{
|
{
|
||||||
auto lock = lockForAlter(RWLockImpl::NO_QUERY, getSettings()->lock_acquire_timeout_for_background_operations);
|
auto table_lock_holder = lockForShare(RWLockImpl::NO_QUERY, getSettings()->lock_acquire_timeout_for_background_operations);
|
||||||
|
auto alter_lock_holder = lockForAlter(getSettings()->lock_acquire_timeout_for_background_operations);
|
||||||
LOG_INFO(log, "Metadata changed in ZooKeeper. Applying changes locally.");
|
LOG_INFO(log, "Metadata changed in ZooKeeper. Applying changes locally.");
|
||||||
|
|
||||||
auto metadata_diff = ReplicatedMergeTreeTableMetadata(*this, getInMemoryMetadataPtr()).checkAndFindDiff(metadata_from_entry);
|
auto metadata_diff = ReplicatedMergeTreeTableMetadata(*this, getInMemoryMetadataPtr()).checkAndFindDiff(metadata_from_entry);
|
||||||
@ -4427,7 +4428,7 @@ PartitionBlockNumbersHolder StorageReplicatedMergeTree::allocateBlockNumbersInAf
|
|||||||
|
|
||||||
|
|
||||||
void StorageReplicatedMergeTree::alter(
|
void StorageReplicatedMergeTree::alter(
|
||||||
const AlterCommands & commands, ContextPtr query_context, TableLockHolder & table_lock_holder)
|
const AlterCommands & commands, ContextPtr query_context, AlterLockHolder & table_lock_holder)
|
||||||
{
|
{
|
||||||
assertNotReadonly();
|
assertNotReadonly();
|
||||||
|
|
||||||
@ -4632,7 +4633,7 @@ void StorageReplicatedMergeTree::alter(
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
table_lock_holder.reset();
|
table_lock_holder.unlock();
|
||||||
|
|
||||||
LOG_DEBUG(log, "Updated shared metadata nodes in ZooKeeper. Waiting for replicas to apply changes.");
|
LOG_DEBUG(log, "Updated shared metadata nodes in ZooKeeper. Waiting for replicas to apply changes.");
|
||||||
waitForLogEntryToBeProcessedIfNecessary(*alter_entry, query_context, "Some replicas doesn't finish metadata alter: ");
|
waitForLogEntryToBeProcessedIfNecessary(*alter_entry, query_context, "Some replicas doesn't finish metadata alter: ");
|
||||||
|
@ -129,7 +129,7 @@ public:
|
|||||||
const Names & deduplicate_by_columns,
|
const Names & deduplicate_by_columns,
|
||||||
ContextPtr query_context) override;
|
ContextPtr query_context) override;
|
||||||
|
|
||||||
void alter(const AlterCommands & commands, ContextPtr query_context, TableLockHolder & table_lock_holder) override;
|
void alter(const AlterCommands & commands, ContextPtr query_context, AlterLockHolder & table_lock_holder) override;
|
||||||
|
|
||||||
void mutate(const MutationCommands & commands, ContextPtr context) override;
|
void mutate(const MutationCommands & commands, ContextPtr context) override;
|
||||||
void waitMutation(const String & znode_name, size_t mutations_sync) const;
|
void waitMutation(const String & znode_name, size_t mutations_sync) const;
|
||||||
|
@ -16,8 +16,6 @@ struct TableExclusiveLockHolder
|
|||||||
private:
|
private:
|
||||||
friend class IStorage;
|
friend class IStorage;
|
||||||
|
|
||||||
/// Order is important.
|
|
||||||
TableLockHolder alter_lock;
|
|
||||||
TableLockHolder drop_lock;
|
TableLockHolder drop_lock;
|
||||||
};
|
};
|
||||||
|
|
||||||
|
Loading…
Reference in New Issue
Block a user