2022-07-26 09:08:55 +00:00
|
|
|
#pragma once
|
|
|
|
|
|
|
|
#include <Interpreters/Context.h>
|
2022-08-21 17:09:36 +00:00
|
|
|
#include <Interpreters/IKeyValueEntity.h>
|
|
|
|
|
2022-07-26 09:08:55 +00:00
|
|
|
#include <QueryPipeline/Pipe.h>
|
|
|
|
#include <Storages/IStorage.h>
|
2022-07-27 13:20:45 +00:00
|
|
|
#include <Storages/StorageInMemoryMetadata.h>
|
2022-08-08 14:09:37 +00:00
|
|
|
#include <Common/PODArray_fwd.h>
|
2022-07-27 13:20:45 +00:00
|
|
|
#include <Common/ZooKeeper/ZooKeeper.h>
|
|
|
|
|
|
|
|
#include <span>
|
2022-07-26 09:08:55 +00:00
|
|
|
|
|
|
|
namespace DB
|
|
|
|
{
|
|
|
|
|
2022-07-27 13:20:45 +00:00
|
|
|
// KV store using (Zoo|CH)Keeper
|
2022-08-21 17:09:36 +00:00
|
|
|
class StorageKeeperMap final : public IStorage, public IKeyValueEntity
|
2022-07-26 09:08:55 +00:00
|
|
|
{
|
|
|
|
public:
|
|
|
|
StorageKeeperMap(
|
2022-07-27 13:20:45 +00:00
|
|
|
ContextPtr context,
|
|
|
|
const StorageID & table_id,
|
|
|
|
const StorageInMemoryMetadata & metadata,
|
2022-08-10 07:24:56 +00:00
|
|
|
bool attach,
|
2022-07-27 13:20:45 +00:00
|
|
|
std::string_view primary_key_,
|
2022-08-10 08:52:36 +00:00
|
|
|
std::string_view root_path_,
|
2022-08-03 13:34:14 +00:00
|
|
|
const std::string & hosts,
|
2022-08-08 09:43:29 +00:00
|
|
|
bool create_missing_root_path,
|
2022-08-10 07:24:56 +00:00
|
|
|
size_t keys_limit,
|
|
|
|
bool remove_existing_data);
|
2022-07-26 09:08:55 +00:00
|
|
|
|
|
|
|
Pipe read(
|
|
|
|
const Names & column_names,
|
|
|
|
const StorageSnapshotPtr & storage_snapshot,
|
|
|
|
SelectQueryInfo & query_info,
|
|
|
|
ContextPtr context,
|
|
|
|
QueryProcessingStage::Enum processed_stage,
|
|
|
|
size_t max_block_size,
|
|
|
|
unsigned num_streams) override;
|
|
|
|
|
2022-07-27 13:20:45 +00:00
|
|
|
SinkToStoragePtr write(const ASTPtr & query, const StorageMetadataPtr & metadata_snapshot, ContextPtr context) override;
|
2022-07-26 09:08:55 +00:00
|
|
|
|
2022-08-10 07:24:56 +00:00
|
|
|
void truncate(const ASTPtr &, const StorageMetadataPtr & , ContextPtr, TableExclusiveLockHolder &) override;
|
|
|
|
void drop() override;
|
|
|
|
|
2022-07-27 13:20:45 +00:00
|
|
|
std::string getName() const override { return "KeeperMap"; }
|
|
|
|
Names getPrimaryKey() const override { return {primary_key}; }
|
2022-07-26 09:08:55 +00:00
|
|
|
|
2022-08-21 17:09:36 +00:00
|
|
|
Chunk getByKeys(const ColumnsWithTypeAndName & keys, PaddedPODArray<UInt8> & null_map, const Names &) const override;
|
2022-07-27 13:20:45 +00:00
|
|
|
Chunk getBySerializedKeys(std::span<const std::string> keys, PaddedPODArray<UInt8> * null_map) const;
|
2022-07-26 09:08:55 +00:00
|
|
|
|
2022-08-21 17:09:36 +00:00
|
|
|
Block getSampleBlock(const Names &) const override;
|
|
|
|
|
2022-07-27 13:20:45 +00:00
|
|
|
bool supportsParallelInsert() const override { return true; }
|
|
|
|
bool supportsIndexForIn() const override { return true; }
|
|
|
|
bool mayBenefitFromIndexForIn(
|
|
|
|
const ASTPtr & node, ContextPtr /*query_context*/, const StorageMetadataPtr & /*metadata_snapshot*/) const override
|
|
|
|
{
|
|
|
|
return node->getColumnName() == primary_key;
|
|
|
|
}
|
|
|
|
|
|
|
|
zkutil::ZooKeeperPtr & getClient() const;
|
2022-07-26 09:08:55 +00:00
|
|
|
const std::string & rootKeeperPath() const;
|
2022-07-27 13:20:45 +00:00
|
|
|
std::string fullPathForKey(std::string_view key) const;
|
2022-07-26 09:08:55 +00:00
|
|
|
|
2022-08-08 09:43:29 +00:00
|
|
|
const std::string & lockPath() const;
|
|
|
|
UInt64 keysLimit() const;
|
|
|
|
|
2022-07-27 13:20:45 +00:00
|
|
|
private:
|
2022-08-10 08:52:36 +00:00
|
|
|
std::string root_path;
|
2022-07-27 13:20:45 +00:00
|
|
|
std::string primary_key;
|
2022-08-08 09:43:29 +00:00
|
|
|
std::string metadata_path;
|
|
|
|
std::string lock_path;
|
|
|
|
UInt64 keys_limit{0};
|
2022-07-27 13:20:45 +00:00
|
|
|
|
|
|
|
mutable zkutil::ZooKeeperPtr zookeeper_client;
|
2022-08-10 08:52:36 +00:00
|
|
|
|
|
|
|
Poco::Logger * log;
|
2022-07-26 09:08:55 +00:00
|
|
|
};
|
|
|
|
|
|
|
|
}
|