This is an automated email from the ASF dual-hosted git repository. morningman pushed a commit to branch branch-3.0 in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-3.0 by this push: new dd9b1b06257 [opt](file-cache) support system table file_cache_statistics #40078 (#42160) dd9b1b06257 is described below commit dd9b1b062571cae4471cd121e15004e98356050b Author: Rayner Chen <morning...@163.com> AuthorDate: Mon Oct 21 17:16:55 2024 +0800 [opt](file-cache) support system table file_cache_statistics #40078 (#42160) cherry pick from #40078 --- be/src/exec/schema_scanner.cpp | 3 + .../schema_file_cache_statistics.cpp | 88 ++++++++++++++++++++++ .../schema_scanner/schema_file_cache_statistics.h | 49 ++++++++++++ be/src/io/cache/block_file_cache.cpp | 71 ++++++++++++++++- be/src/io/cache/block_file_cache.h | 18 ++++- be/src/io/cache/block_file_cache_factory.cpp | 21 ++++++ be/src/io/cache/block_file_cache_factory.h | 6 ++ .../org/apache/doris/analysis/SchemaTableType.java | 2 + .../java/org/apache/doris/catalog/SchemaTable.java | 9 +++ .../planner/BackendPartitionedSchemaScanNode.java | 2 + .../cache/test_file_cache_statistics.out | 4 + .../cache/test_file_cache_statistics.groovy | 59 +++++++++++++++ 12 files changed, 326 insertions(+), 6 deletions(-) diff --git a/be/src/exec/schema_scanner.cpp b/be/src/exec/schema_scanner.cpp index 90140e748f5..1b329f76ff6 100644 --- a/be/src/exec/schema_scanner.cpp +++ b/be/src/exec/schema_scanner.cpp @@ -33,6 +33,7 @@ #include "exec/schema_scanner/schema_collations_scanner.h" #include "exec/schema_scanner/schema_columns_scanner.h" #include "exec/schema_scanner/schema_dummy_scanner.h" +#include "exec/schema_scanner/schema_file_cache_statistics.h" #include "exec/schema_scanner/schema_files_scanner.h" #include "exec/schema_scanner/schema_metadata_name_ids_scanner.h" #include "exec/schema_scanner/schema_partitions_scanner.h" @@ -241,6 +242,8 @@ std::unique_ptr<SchemaScanner> SchemaScanner::create(TSchemaTableType::type type return SchemaBackendWorkloadGroupResourceUsage::create_unique(); case TSchemaTableType::SCH_TABLE_PROPERTIES: return SchemaTablePropertiesScanner::create_unique(); + case TSchemaTableType::SCH_FILE_CACHE_STATISTICS: + return SchemaFileCacheStatisticsScanner::create_unique(); case TSchemaTableType::SCH_CATALOG_META_CACHE_STATISTICS: return SchemaCatalogMetaCacheStatsScanner::create_unique(); default: diff --git a/be/src/exec/schema_scanner/schema_file_cache_statistics.cpp b/be/src/exec/schema_scanner/schema_file_cache_statistics.cpp new file mode 100644 index 00000000000..ecad274d218 --- /dev/null +++ b/be/src/exec/schema_scanner/schema_file_cache_statistics.cpp @@ -0,0 +1,88 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +#include "exec/schema_scanner/schema_file_cache_statistics.h" + +#include "io/cache/block_file_cache_factory.h" +#include "runtime/exec_env.h" +#include "runtime/runtime_state.h" +#include "vec/common/string_ref.h" +#include "vec/core/block.h" +#include "vec/data_types/data_type_factory.hpp" + +namespace doris { + +std::vector<SchemaScanner::ColumnDesc> SchemaFileCacheStatisticsScanner::_s_tbls_columns = { + // name, type, size + {"BE_ID", TYPE_BIGINT, sizeof(int64_t), false}, + {"BE_IP", TYPE_STRING, sizeof(StringRef), false}, + {"CACHE_PATH", TYPE_STRING, sizeof(StringRef), false}, + {"METRIC_NAME", TYPE_STRING, sizeof(StringRef), false}, + {"METRIC_VALUE", TYPE_STRING, sizeof(StringRef), false}}; + +SchemaFileCacheStatisticsScanner::SchemaFileCacheStatisticsScanner() + : SchemaScanner(_s_tbls_columns, TSchemaTableType::SCH_FILE_CACHE_STATISTICS) {} + +SchemaFileCacheStatisticsScanner::~SchemaFileCacheStatisticsScanner() {} + +Status SchemaFileCacheStatisticsScanner::start(RuntimeState* state) { + _block_rows_limit = state->batch_size(); + return Status::OK(); +} + +Status SchemaFileCacheStatisticsScanner::get_next_block_internal(vectorized::Block* block, + bool* eos) { + if (!_is_init) { + return Status::InternalError("Used before initialized."); + } + + if (nullptr == block || nullptr == eos) { + return Status::InternalError("input pointer is nullptr."); + } + + if (_stats_block == nullptr) { + _stats_block = vectorized::Block::create_unique(); + + for (int i = 0; i < _s_tbls_columns.size(); ++i) { + TypeDescriptor descriptor(_s_tbls_columns[i].type); + auto data_type = + vectorized::DataTypeFactory::instance().create_data_type(descriptor, true); + _stats_block->insert(vectorized::ColumnWithTypeAndName( + data_type->create_column(), data_type, _s_tbls_columns[i].name)); + } + + _stats_block->reserve(_block_rows_limit); + + ExecEnv::GetInstance()->file_cache_factory()->get_cache_stats_block(_stats_block.get()); + _total_rows = _stats_block->rows(); + } + + if (_row_idx == _total_rows) { + *eos = true; + return Status::OK(); + } + + int current_batch_rows = std::min(_block_rows_limit, _total_rows - _row_idx); + vectorized::MutableBlock mblock = vectorized::MutableBlock::build_mutable_block(block); + RETURN_IF_ERROR(mblock.add_rows(_stats_block.get(), _row_idx, current_batch_rows)); + _row_idx += current_batch_rows; + + *eos = _row_idx == _total_rows; + return Status::OK(); +} + +} // namespace doris diff --git a/be/src/exec/schema_scanner/schema_file_cache_statistics.h b/be/src/exec/schema_scanner/schema_file_cache_statistics.h new file mode 100644 index 00000000000..96c6aa9028f --- /dev/null +++ b/be/src/exec/schema_scanner/schema_file_cache_statistics.h @@ -0,0 +1,49 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +#pragma once + +#include <vector> + +#include "common/status.h" +#include "exec/schema_scanner.h" + +namespace doris { +class RuntimeState; +namespace vectorized { +class Block; +} // namespace vectorized + +class SchemaFileCacheStatisticsScanner : public SchemaScanner { + ENABLE_FACTORY_CREATOR(SchemaFileCacheStatisticsScanner); + +public: + SchemaFileCacheStatisticsScanner(); + ~SchemaFileCacheStatisticsScanner() override; + + Status start(RuntimeState* state) override; + Status get_next_block_internal(vectorized::Block* block, bool* eos) override; + + static std::vector<SchemaScanner::ColumnDesc> _s_tbls_columns; + +private: + int _block_rows_limit = 4096; + int _row_idx = 0; + int _total_rows = 0; + std::unique_ptr<vectorized::Block> _stats_block = nullptr; +}; +}; // namespace doris diff --git a/be/src/io/cache/block_file_cache.cpp b/be/src/io/cache/block_file_cache.cpp index 2ff374442d1..321d9d73d89 100644 --- a/be/src/io/cache/block_file_cache.cpp +++ b/be/src/io/cache/block_file_cache.cpp @@ -83,6 +83,30 @@ BlockFileCache::BlockFileCache(const std::string& cache_base_path, _total_evict_size_metrics = std::make_shared<bvar::Adder<size_t>>( _cache_base_path.c_str(), "file_cache_total_evict_size"); + _num_read_blocks = std::make_shared<bvar::Adder<size_t>>(_cache_base_path.c_str(), + "file_cache_num_read_blocks"); + _num_hit_blocks = std::make_shared<bvar::Adder<size_t>>(_cache_base_path.c_str(), + "file_cache_num_hit_blocks"); + _num_removed_blocks = std::make_shared<bvar::Adder<size_t>>(_cache_base_path.c_str(), + "file_cache_num_removed_blocks"); + + _num_hit_blocks_5m = std::make_shared<bvar::Window<bvar::Adder<size_t>>>( + _cache_base_path.c_str(), "file_cache_num_hit_blocks_5m", _num_hit_blocks.get(), 300); + _num_read_blocks_5m = std::make_shared<bvar::Window<bvar::Adder<size_t>>>( + _cache_base_path.c_str(), "file_cache_num_read_blocks_5m", _num_read_blocks.get(), 300); + _num_hit_blocks_1h = std::make_shared<bvar::Window<bvar::Adder<size_t>>>( + _cache_base_path.c_str(), "file_cache_num_hit_blocks_1h", _num_hit_blocks.get(), 3600); + _num_read_blocks_1h = std::make_shared<bvar::Window<bvar::Adder<size_t>>>( + _cache_base_path.c_str(), "file_cache_num_read_blocks_1h", _num_read_blocks.get(), + 3600); + + _hit_ratio = std::make_shared<bvar::Status<double>>(_cache_base_path.c_str(), + "file_cache_hit_ratio", 0.0); + _hit_ratio_5m = std::make_shared<bvar::Status<double>>(_cache_base_path.c_str(), + "file_cache_hit_ratio_5m", 0.0); + _hit_ratio_1h = std::make_shared<bvar::Status<double>>(_cache_base_path.c_str(), + "file_cache_hit_ratio_1h", 0.0); + _disposable_queue = LRUQueue(cache_settings.disposable_queue_size, cache_settings.disposable_queue_elements, 60 * 60); _index_queue = LRUQueue(cache_settings.index_queue_size, cache_settings.index_queue_elements, @@ -667,10 +691,10 @@ FileBlocksHolder BlockFileCache::get_or_set(const UInt128Wrapper& hash, size_t o fill_holes_with_empty_file_blocks(file_blocks, hash, context, range, cache_lock); } DCHECK(!file_blocks.empty()); - _num_read_blocks += file_blocks.size(); + *_num_read_blocks << file_blocks.size(); for (auto& block : file_blocks) { if (block->state() == FileBlock::State::DOWNLOADED) { - _num_hit_blocks++; + *_num_hit_blocks << 1; } } return FileBlocksHolder(std::move(file_blocks)); @@ -1271,7 +1295,7 @@ void BlockFileCache::remove(FileBlockSPtr file_block, T& cache_lock, U& block_lo if (offsets.empty()) { _files.erase(hash); } - _num_removed_blocks++; + *_num_removed_blocks << 1; } size_t BlockFileCache::get_used_cache_size(FileCacheType cache_type) const { @@ -1600,6 +1624,19 @@ void BlockFileCache::run_background_operation() { _disposable_queue.get_capacity(cache_lock)); _cur_disposable_queue_element_count_metrics->set_value( _disposable_queue.get_elements_num(cache_lock)); + + if (_num_read_blocks->get_value() > 0) { + _hit_ratio->set_value((double)_num_hit_blocks->get_value() / + _num_read_blocks->get_value()); + } + if (_num_read_blocks_5m->get_value() > 0) { + _hit_ratio_5m->set_value((double)_num_hit_blocks_5m->get_value() / + _num_read_blocks_5m->get_value()); + } + if (_num_read_blocks_1h->get_value() > 0) { + _hit_ratio_1h->set_value((double)_num_hit_blocks_1h->get_value() / + _num_read_blocks_1h->get_value()); + } } } @@ -1802,6 +1839,34 @@ void BlockFileCache::update_ttl_atime(const UInt128Wrapper& hash) { }; } +std::map<std::string, double> BlockFileCache::get_stats() { + std::map<std::string, double> stats; + stats["hits_ratio"] = (double)_hit_ratio->get_value(); + stats["hits_ratio_5m"] = (double)_hit_ratio_5m->get_value(); + stats["hits_ratio_1h"] = (double)_hit_ratio_1h->get_value(); + + stats["index_queue_max_size"] = (double)_index_queue.get_max_size(); + stats["index_queue_curr_size"] = (double)_cur_index_queue_element_count_metrics->get_value(); + stats["index_queue_max_elements"] = (double)_index_queue.get_max_element_size(); + stats["index_queue_curr_elements"] = + (double)_cur_index_queue_element_count_metrics->get_value(); + + stats["normal_queue_max_size"] = (double)_normal_queue.get_max_size(); + stats["normal_queue_curr_size"] = (double)_cur_normal_queue_element_count_metrics->get_value(); + stats["normal_queue_max_elements"] = (double)_normal_queue.get_max_element_size(); + stats["normal_queue_curr_elements"] = + (double)_cur_normal_queue_element_count_metrics->get_value(); + + stats["disposable_queue_max_size"] = (double)_disposable_queue.get_max_size(); + stats["disposable_queue_curr_size"] = + (double)_cur_disposable_queue_element_count_metrics->get_value(); + stats["disposable_queue_max_elements"] = (double)_disposable_queue.get_max_element_size(); + stats["disposable_queue_curr_elements"] = + (double)_cur_disposable_queue_element_count_metrics->get_value(); + + return stats; +} + template void BlockFileCache::remove(FileBlockSPtr file_block, std::lock_guard<std::mutex>& cache_lock, std::lock_guard<std::mutex>& block_lock); diff --git a/be/src/io/cache/block_file_cache.h b/be/src/io/cache/block_file_cache.h index c0a2bce76b1..ac30e2411fa 100644 --- a/be/src/io/cache/block_file_cache.h +++ b/be/src/io/cache/block_file_cache.h @@ -143,6 +143,8 @@ public: void update_ttl_atime(const UInt128Wrapper& hash); + std::map<std::string, double> get_stats(); + class LRUQueue { public: LRUQueue() = default; @@ -445,9 +447,6 @@ private: LRUQueue _ttl_queue; // metrics - size_t _num_read_blocks = 0; - size_t _num_hit_blocks = 0; - size_t _num_removed_blocks = 0; std::shared_ptr<bvar::Status<size_t>> _cur_cache_size_metrics; std::shared_ptr<bvar::Status<size_t>> _cur_ttl_cache_size_metrics; std::shared_ptr<bvar::Status<size_t>> _cur_ttl_cache_lru_queue_cache_size_metrics; @@ -460,6 +459,19 @@ private: std::shared_ptr<bvar::Status<size_t>> _cur_disposable_queue_cache_size_metrics; std::array<std::shared_ptr<bvar::Adder<size_t>>, 4> _queue_evict_size_metrics; std::shared_ptr<bvar::Adder<size_t>> _total_evict_size_metrics; + + std::shared_ptr<bvar::Window<bvar::Adder<size_t>>> _num_hit_blocks_5m; + std::shared_ptr<bvar::Window<bvar::Adder<size_t>>> _num_read_blocks_5m; + std::shared_ptr<bvar::Window<bvar::Adder<size_t>>> _num_hit_blocks_1h; + std::shared_ptr<bvar::Window<bvar::Adder<size_t>>> _num_read_blocks_1h; + + std::shared_ptr<bvar::Adder<size_t>> _num_read_blocks; + std::shared_ptr<bvar::Adder<size_t>> _num_hit_blocks; + std::shared_ptr<bvar::Adder<size_t>> _num_removed_blocks; + + std::shared_ptr<bvar::Status<double>> _hit_ratio; + std::shared_ptr<bvar::Status<double>> _hit_ratio_5m; + std::shared_ptr<bvar::Status<double>> _hit_ratio_1h; }; } // namespace doris::io diff --git a/be/src/io/cache/block_file_cache_factory.cpp b/be/src/io/cache/block_file_cache_factory.cpp index ac16bbefa58..8370962ddd5 100644 --- a/be/src/io/cache/block_file_cache_factory.cpp +++ b/be/src/io/cache/block_file_cache_factory.cpp @@ -32,9 +32,12 @@ #include <utility> #include "common/config.h" +#include "exec/schema_scanner/schema_scanner_helper.h" #include "io/cache/file_cache_common.h" #include "io/fs/local_file_system.h" #include "runtime/exec_env.h" +#include "service/backend_options.h" +#include "vec/core/block.h" namespace doris { class TUniqueId; @@ -169,5 +172,23 @@ std::string FileCacheFactory::reset_capacity(const std::string& path, int64_t ne return "Unknown the cache path " + path; } +void FileCacheFactory::get_cache_stats_block(vectorized::Block* block) { + // std::shared_lock<std::shared_mutex> read_lock(_qs_ctx_map_lock); + TBackend be = BackendOptions::get_local_backend(); + int64_t be_id = be.id; + std::string be_ip = be.host; + for (auto& cache : _caches) { + std::map<std::string, double> stats = cache->get_stats(); + for (auto& [k, v] : stats) { + SchemaScannerHelper::insert_int64_value(0, be_id, block); // be id + SchemaScannerHelper::insert_string_value(1, be_ip, block); // be ip + SchemaScannerHelper::insert_string_value(2, cache->get_base_path(), + block); // cache path + SchemaScannerHelper::insert_string_value(3, k, block); // metric name + SchemaScannerHelper::insert_string_value(4, std::to_string(v), block); // metric value + } + } +} + } // namespace io } // namespace doris diff --git a/be/src/io/cache/block_file_cache_factory.h b/be/src/io/cache/block_file_cache_factory.h index d7b710876ce..12714fd2087 100644 --- a/be/src/io/cache/block_file_cache_factory.h +++ b/be/src/io/cache/block_file_cache_factory.h @@ -32,6 +32,10 @@ namespace doris { class TUniqueId; +namespace vectorized { +class Block; +} // namespace vectorized + namespace io { /** @@ -82,6 +86,8 @@ public: */ std::string reset_capacity(const std::string& path, int64_t new_capacity); + void get_cache_stats_block(vectorized::Block* block); + FileCacheFactory() = default; FileCacheFactory& operator=(const FileCacheFactory&) = delete; FileCacheFactory(const FileCacheFactory&) = delete; diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java index 7fcb61e3d2d..f5bb506ffa3 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java @@ -85,6 +85,8 @@ public enum SchemaTableType { "WORKLOAD_GROUP_RESOURCE_USAGE", TSchemaTableType.SCH_WORKLOAD_GROUP_RESOURCE_USAGE), SCH_TABLE_PROPERTIES("TABLE_PROPERTIES", "TABLE_PROPERTIES", TSchemaTableType.SCH_TABLE_PROPERTIES), + SCH_FILE_CACHE_STATISTICS("FILE_CACHE_STATISTICS", "FILE_CACHE_STATISTICS", + TSchemaTableType.SCH_FILE_CACHE_STATISTICS), SCH_CATALOG_META_CACHE_STATISTICS("CATALOG_META_CACHE_STATISTICS", "CATALOG_META_CACHE_STATISTICS", TSchemaTableType.SCH_CATALOG_META_CACHE_STATISTICS); diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java b/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java index 229141787b9..e022cb85296 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java +++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java @@ -558,6 +558,15 @@ public class SchemaTable extends Table { .column("REMOTE_SCAN_BYTES_PER_SECOND", ScalarType.createType(PrimitiveType.BIGINT)) .build()) ) + .put("file_cache_statistics", + new SchemaTable(SystemIdGenerator.getNextId(), "file_cache_statistics", TableType.SCHEMA, + builder().column("BE_ID", ScalarType.createType(PrimitiveType.BIGINT)) + .column("BE_IP", ScalarType.createStringType()) + .column("CACHE_PATH", ScalarType.createStringType()) + .column("METRIC_NAME", ScalarType.createStringType()) + .column("METRIC_VALUE", ScalarType.createStringType()) + .build()) + ) .put("catalog_meta_cache_statistics", new SchemaTable(SystemIdGenerator.getNextId(), "catalog_meta_cache_statistics", TableType.SCHEMA, builder().column("CATALOG_NAME", ScalarType.createStringType()) diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java b/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java index cf5c85e98b7..207197ec06e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java +++ b/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java @@ -67,6 +67,8 @@ public class BackendPartitionedSchemaScanNode extends SchemaScanNode { BACKEND_TABLE.add("backend_active_tasks"); BACKEND_TABLE.add("workload_group_resource_usage"); BEACKEND_ID_COLUMN_SET.add("be_id"); + + BACKEND_TABLE.add("file_cache_statistics"); } public static boolean isBackendPartitionedSchemaTable(String tableName) { diff --git a/regression-test/data/external_table_p0/cache/test_file_cache_statistics.out b/regression-test/data/external_table_p0/cache/test_file_cache_statistics.out new file mode 100644 index 00000000000..39444cdeb5c --- /dev/null +++ b/regression-test/data/external_table_p0/cache/test_file_cache_statistics.out @@ -0,0 +1,4 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !2 -- +1 7706 1 155190 17.00 21168.23 0.04 0.02 N O 1996-03-13 1996-02-12 1996-03-22 DELIVER IN PERSON TRUCK egular courts above the cn beijing + diff --git a/regression-test/suites/external_table_p0/cache/test_file_cache_statistics.groovy b/regression-test/suites/external_table_p0/cache/test_file_cache_statistics.groovy new file mode 100644 index 00000000000..dfceba90cec --- /dev/null +++ b/regression-test/suites/external_table_p0/cache/test_file_cache_statistics.groovy @@ -0,0 +1,59 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you 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. + +import java.util.concurrent.TimeUnit; +import org.awaitility.Awaitility; + +suite("test_file_cache_statistics", "external_docker,hive,external_docker_hive,p0,external") { + String enabled = context.config.otherConfigs.get("enableHiveTest") + if (enabled == null || !enabled.equalsIgnoreCase("true")) { + logger.info("diable Hive test.") + return; + } + + String catalog_name = "test_file_cache_statistics" + String ex_db_name = "`default`" + String externalEnvIp = context.config.otherConfigs.get("externalEnvIp") + String hms_port = context.config.otherConfigs.get(hivePrefix + "HmsPort") + String hdfs_port = context.config.otherConfigs.get(hivePrefix + "HdfsPort") + + sql """set enable_file_cache=true""" + sql """drop catalog if exists ${catalog_name} """ + + sql """CREATE CATALOG ${catalog_name} PROPERTIES ( + 'type'='hms', + 'hive.metastore.uris' = 'thrift://${externalEnvIp}:${hms_port}', + 'hadoop.username' = 'hive' + );""" + + sql """switch ${catalog_name}""" + + order_qt_2 """select * from ${catalog_name}.${ex_db_name}.parquet_partition_table order by l_orderkey limit 1;""" + // brpc metrics will be updated at most 20 seconds + Awaitility.await().atMost(30, TimeUnit.SECONDS).pollInterval(1, TimeUnit.SECONDS).until{ + result = sql """select METRIC_VALUE from information_schema.file_cache_statistics where METRIC_NAME like "%hits_ratio%" order by METRIC_VALUE limit 1;""" + logger.info("result " + result) + if (result.size() == 0) { + return false; + } + if (Double.valueOf(result[0][0]) > 0) { + return true; + } + return false; + } +} + --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org