You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by mo...@apache.org on 2021/01/04 04:19:44 UTC

[incubator-doris] branch master updated: [Optimize][Cache]Implementation of Separated Page Cache (#5008)

This is an automated email from the ASF dual-hosted git repository.

morningman pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git


The following commit(s) were added to refs/heads/master by this push:
     new 6c098e4  [Optimize][Cache]Implementation of Separated Page Cache (#5008)
6c098e4 is described below

commit 6c098e45fc33342da6bb47dbdc34dda1e40a3cec
Author: Skysheepwang <wa...@163.com>
AuthorDate: Mon Jan 4 12:19:24 2021 +0800

    [Optimize][Cache]Implementation of Separated Page Cache (#5008)
    
    #4995
    **Implementation of Separated Page Cache**
    - Add config "index_page_cache_ratio" to set the ratio of capacity of index page cache
    - Change the member of StoragePageCache to maintain two type of cache
    - Change the interface of StoragePageCache for selecting type of cache
    - Change the usage of page cache in read_and_decompress_page in page_io.cpp
      - add page type as argument
      - check if current page type is available in StoragePageCache (cover the situation of ratio == 0 or 1)
    - Add type as argument in superior call of read_and_decompress_page
    - Change Unit Test
---
 be/src/common/config.h                             |   3 +
 be/src/olap/page_cache.cpp                         |  33 ++--
 be/src/olap/page_cache.h                           |  33 +++-
 be/src/olap/rowset/segment_v2/column_reader.cpp    |   3 +
 be/src/olap/rowset/segment_v2/column_reader.h      |   4 +
 .../rowset/segment_v2/indexed_column_reader.cpp    |   7 +-
 .../olap/rowset/segment_v2/indexed_column_reader.h |   2 +-
 .../olap/rowset/segment_v2/ordinal_page_index.cpp  |   1 +
 be/src/olap/rowset/segment_v2/page_io.cpp          |   6 +-
 be/src/olap/rowset/segment_v2/page_io.h            |   4 +
 be/src/olap/rowset/segment_v2/segment.cpp          |   1 +
 be/src/runtime/exec_env_init.cpp                   |   3 +-
 be/test/olap/page_cache_test.cpp                   | 180 +++++++++++++++++++--
 be/test/olap/rowset/beta_rowset_test.cpp           |   2 +-
 be/test/olap/rowset/rowset_converter_test.cpp      |   2 +-
 .../olap/rowset/segment_v2/bitmap_index_test.cpp   |   2 +-
 .../bloom_filter_index_reader_writer_test.cpp      |   2 +-
 .../segment_v2/column_reader_writer_test.cpp       |   2 +-
 .../rowset/segment_v2/ordinal_page_index_test.cpp  |   2 +-
 be/test/olap/rowset/segment_v2/segment_test.cpp    |   2 +-
 .../olap/rowset/segment_v2/zone_map_index_test.cpp |   2 +-
 docs/en/administrator-guide/config/be_config.md    |   5 +
 docs/zh-CN/administrator-guide/config/be_config.md |   5 +
 23 files changed, 266 insertions(+), 40 deletions(-)

diff --git a/be/src/common/config.h b/be/src/common/config.h
index 6bef85a..4c5df68 100644
--- a/be/src/common/config.h
+++ b/be/src/common/config.h
@@ -260,6 +260,9 @@ CONF_Int64(index_stream_cache_capacity, "10737418240");
 
 // Cache for storage page size
 CONF_String(storage_page_cache_limit, "20%");
+// Percentage for index page cache
+// all storage page cache will be divided into data_page_cache and index_page_cache
+CONF_Int32(index_page_cache_percentage, "10");
 // whether to disable page cache feature in storage
 CONF_Bool(disable_storage_page_cache, "false");
 
diff --git a/be/src/olap/page_cache.cpp b/be/src/olap/page_cache.cpp
index aceb663..787ef1c 100644
--- a/be/src/olap/page_cache.cpp
+++ b/be/src/olap/page_cache.cpp
@@ -21,26 +21,38 @@ namespace doris {
 
 StoragePageCache* StoragePageCache::_s_instance = nullptr;
 
-void StoragePageCache::create_global_cache(size_t capacity) {
+void StoragePageCache::create_global_cache(size_t capacity, int32_t index_cache_percentage) {
     DCHECK(_s_instance == nullptr);
-    static StoragePageCache instance(capacity);
+    static StoragePageCache instance(capacity, index_cache_percentage);
     _s_instance = &instance;
 }
 
-StoragePageCache::StoragePageCache(size_t capacity)
-        : _cache(new_lru_cache("StoragePageCache", capacity)) {}
+StoragePageCache::StoragePageCache(size_t capacity, int32_t index_cache_percentage)
+        : _index_cache_percentage(index_cache_percentage) {
+    if (index_cache_percentage == 0) {
+        _data_page_cache = std::unique_ptr<Cache>(new_lru_cache("DataPageCache", capacity));
+    } else if (index_cache_percentage == 100) {
+        _index_page_cache = std::unique_ptr<Cache>(new_lru_cache("IndexPageCache", capacity));
+    } else if (index_cache_percentage > 0 && index_cache_percentage < 100) {
+        _data_page_cache = std::unique_ptr<Cache>(new_lru_cache("DataPageCache", capacity * (100 - index_cache_percentage) / 100));
+        _index_page_cache = std::unique_ptr<Cache>(new_lru_cache("IndexPageCache", capacity * index_cache_percentage / 100));
+    } else {
+        CHECK(false) << "invalid index page cache percentage";
+    }
+}
 
-bool StoragePageCache::lookup(const CacheKey& key, PageCacheHandle* handle) {
-    auto lru_handle = _cache->lookup(key.encode());
+bool StoragePageCache::lookup(const CacheKey& key, PageCacheHandle* handle, segment_v2::PageTypePB page_type) {
+    auto cache = _get_page_cache(page_type);
+    auto lru_handle = cache->lookup(key.encode());
     if (lru_handle == nullptr) {
         return false;
     }
-    *handle = PageCacheHandle(_cache.get(), lru_handle);
+    *handle = PageCacheHandle(cache, lru_handle);
     return true;
 }
 
 void StoragePageCache::insert(const CacheKey& key, const Slice& data, PageCacheHandle* handle,
-                              bool in_memory) {
+                              segment_v2::PageTypePB page_type, bool in_memory) {
     auto deleter = [](const doris::CacheKey& key, void* value) { delete[](uint8_t*) value; };
 
     CachePriority priority = CachePriority::NORMAL;
@@ -48,8 +60,9 @@ void StoragePageCache::insert(const CacheKey& key, const Slice& data, PageCacheH
         priority = CachePriority::DURABLE;
     }
 
-    auto lru_handle = _cache->insert(key.encode(), data.data, data.size, deleter, priority);
-    *handle = PageCacheHandle(_cache.get(), lru_handle);
+    auto cache = _get_page_cache(page_type);
+    auto lru_handle = cache->insert(key.encode(), data.data, data.size, deleter, priority);
+    *handle = PageCacheHandle(cache, lru_handle);
 }
 
 } // namespace doris
diff --git a/be/src/olap/page_cache.h b/be/src/olap/page_cache.h
index e48a2c9..aa3a600 100644
--- a/be/src/olap/page_cache.h
+++ b/be/src/olap/page_cache.h
@@ -23,6 +23,7 @@
 
 #include "gutil/macros.h" // for DISALLOW_COPY_AND_ASSIGN
 #include "olap/lru_cache.h"
+#include "gen_cpp/segment_v2.pb.h" // for cache allocation
 
 namespace doris {
 
@@ -53,13 +54,13 @@ public:
     };
 
     // Create global instance of this class
-    static void create_global_cache(size_t capacity);
+    static void create_global_cache(size_t capacity, int32_t index_cache_percentage);
 
     // Return global instance.
     // Client should call create_global_cache before.
     static StoragePageCache* instance() { return _s_instance; }
 
-    StoragePageCache(size_t capacity);
+    StoragePageCache(size_t capacity, int32_t index_cache_percentage);
 
     // Lookup the given page in the cache.
     //
@@ -67,8 +68,10 @@ public:
     // PageCacheHandle will release cache entry to cache when it
     // destructs.
     //
+    // Cache type selection is determined by page_type argument
+    //
     // Return true if entry is found, otherwise return false.
-    bool lookup(const CacheKey& key, PageCacheHandle* handle);
+    bool lookup(const CacheKey& key, PageCacheHandle* handle, segment_v2::PageTypePB page_type);
 
     // Insert a page with key into this cache.
     // Given handle will be set to valid reference.
@@ -76,13 +79,33 @@ public:
     // concurrently, this function can assure that only one page is cached.
     // The in_memory page will have higher priority.
     void insert(const CacheKey& key, const Slice& data, PageCacheHandle* handle,
-                bool in_memory = false);
+                segment_v2::PageTypePB page_type, bool in_memory = false);
+
+    // Page cache available check.
+    // When percentage is set to 0 or 100, the index or data cache will not be allocated.
+    bool is_cache_available(segment_v2::PageTypePB page_type) {
+        return _get_page_cache(page_type) != nullptr;
+    }
 
 private:
     StoragePageCache();
     static StoragePageCache* _s_instance;
 
-    std::unique_ptr<Cache> _cache = nullptr;
+    int32_t _index_cache_percentage = 0;
+    std::unique_ptr<Cache> _data_page_cache = nullptr;
+    std::unique_ptr<Cache> _index_page_cache = nullptr;
+
+    Cache* _get_page_cache(segment_v2::PageTypePB page_type) {
+        switch (page_type)
+        {
+        case segment_v2::DATA_PAGE:
+            return _data_page_cache.get();
+        case segment_v2::INDEX_PAGE:
+            return _index_page_cache.get();
+        default:
+            return nullptr;
+        }
+    }
 };
 
 // A handle for StoragePageCache entry. This class make it easy to handle
diff --git a/be/src/olap/rowset/segment_v2/column_reader.cpp b/be/src/olap/rowset/segment_v2/column_reader.cpp
index b2d382e..cfe213f 100644
--- a/be/src/olap/rowset/segment_v2/column_reader.cpp
+++ b/be/src/olap/rowset/segment_v2/column_reader.cpp
@@ -130,6 +130,7 @@ Status ColumnReader::read_page(const ColumnIteratorOptions& iter_opts, const Pag
     opts.verify_checksum = _opts.verify_checksum;
     opts.use_page_cache = iter_opts.use_page_cache;
     opts.kept_in_memory = _opts.kept_in_memory;
+    opts.type = iter_opts.type;
 
     return PageIO::read_and_decompress_page(opts, handle, page_body, footer);
 }
@@ -569,6 +570,7 @@ Status FileColumnIterator::_read_data_page(const OrdinalPageIndexIterator& iter)
     PageHandle handle;
     Slice page_body;
     PageFooterPB footer;
+    _opts.type = DATA_PAGE;
     RETURN_IF_ERROR(_reader->read_page(_opts, iter.page(), &handle, &page_body, &footer));
     // parse data page
     RETURN_IF_ERROR(ParsedPage::create(std::move(handle), page_body, footer.data_page_footer(),
@@ -587,6 +589,7 @@ Status FileColumnIterator::_read_data_page(const OrdinalPageIndexIterator& iter)
                 // read dictionary page
                 Slice dict_data;
                 PageFooterPB dict_footer;
+                _opts.type = INDEX_PAGE;
                 RETURN_IF_ERROR(_reader->read_page(_opts, _reader->get_dict_page_pointer(),
                                                    &_dict_page_handle, &dict_data, &dict_footer));
                 // ignore dict_footer.dict_page_footer().encoding() due to only
diff --git a/be/src/olap/rowset/segment_v2/column_reader.h b/be/src/olap/rowset/segment_v2/column_reader.h
index c6a8b33..7384fa9 100644
--- a/be/src/olap/rowset/segment_v2/column_reader.h
+++ b/be/src/olap/rowset/segment_v2/column_reader.h
@@ -67,6 +67,10 @@ struct ColumnIteratorOptions {
     // reader statistics
     OlapReaderStatistics* stats = nullptr;
     bool use_page_cache = false;
+    // for page cache allocation
+    // page types are divided into DATA_PAGE & INDEX_PAGE
+    // INDEX_PAGE including index_page, dict_page and short_key_page
+    PageTypePB type;
 
     void sanity_check() const {
         CHECK_NOTNULL(rblock);
diff --git a/be/src/olap/rowset/segment_v2/indexed_column_reader.cpp b/be/src/olap/rowset/segment_v2/indexed_column_reader.cpp
index 427bb9e..1ac6692 100644
--- a/be/src/olap/rowset/segment_v2/indexed_column_reader.cpp
+++ b/be/src/olap/rowset/segment_v2/indexed_column_reader.cpp
@@ -72,13 +72,13 @@ Status IndexedColumnReader::load_index_page(fs::ReadableBlock* rblock, const Pag
                                             PageHandle* handle, IndexPageReader* reader) {
     Slice body;
     PageFooterPB footer;
-    RETURN_IF_ERROR(read_page(rblock, PagePointer(pp), handle, &body, &footer));
+    RETURN_IF_ERROR(read_page(rblock, PagePointer(pp), handle, &body, &footer, INDEX_PAGE));
     RETURN_IF_ERROR(reader->parse(body, footer.index_page_footer()));
     return Status::OK();
 }
 
 Status IndexedColumnReader::read_page(fs::ReadableBlock* rblock, const PagePointer& pp,
-                                      PageHandle* handle, Slice* body, PageFooterPB* footer) const {
+                                      PageHandle* handle, Slice* body, PageFooterPB* footer, PageTypePB type) const {
     PageReadOptions opts;
     opts.rblock = rblock;
     opts.page_pointer = pp;
@@ -87,6 +87,7 @@ Status IndexedColumnReader::read_page(fs::ReadableBlock* rblock, const PagePoint
     opts.stats = &tmp_stats;
     opts.use_page_cache = _use_page_cache;
     opts.kept_in_memory = _kept_in_memory;
+    opts.type = type;
 
     return PageIO::read_and_decompress_page(opts, handle, body, footer);
 }
@@ -97,7 +98,7 @@ Status IndexedColumnIterator::_read_data_page(const PagePointer& pp) {
     PageHandle handle;
     Slice body;
     PageFooterPB footer;
-    RETURN_IF_ERROR(_reader->read_page(_rblock.get(), pp, &handle, &body, &footer));
+    RETURN_IF_ERROR(_reader->read_page(_rblock.get(), pp, &handle, &body, &footer, DATA_PAGE));
     // parse data page
     // note that page_index is not used in IndexedColumnIterator, so we pass 0
     return ParsedPage::create(std::move(handle), body, footer.data_page_footer(),
diff --git a/be/src/olap/rowset/segment_v2/indexed_column_reader.h b/be/src/olap/rowset/segment_v2/indexed_column_reader.h
index 1951cb1..1cd42d8 100644
--- a/be/src/olap/rowset/segment_v2/indexed_column_reader.h
+++ b/be/src/olap/rowset/segment_v2/indexed_column_reader.h
@@ -51,7 +51,7 @@ public:
 
     // read a page specified by `pp' from `file' into `handle'
     Status read_page(fs::ReadableBlock* rblock, const PagePointer& pp, PageHandle* handle,
-                     Slice* body, PageFooterPB* footer) const;
+                     Slice* body, PageFooterPB* footer, PageTypePB type) const;
 
     int64_t num_values() const { return _num_values; }
     const EncodingInfo* encoding_info() const { return _encoding_info; }
diff --git a/be/src/olap/rowset/segment_v2/ordinal_page_index.cpp b/be/src/olap/rowset/segment_v2/ordinal_page_index.cpp
index 8d3a63b..6c4d94c 100644
--- a/be/src/olap/rowset/segment_v2/ordinal_page_index.cpp
+++ b/be/src/olap/rowset/segment_v2/ordinal_page_index.cpp
@@ -80,6 +80,7 @@ Status OrdinalIndexReader::load(bool use_page_cache, bool kept_in_memory) {
     opts.stats = &tmp_stats;
     opts.use_page_cache = use_page_cache;
     opts.kept_in_memory = kept_in_memory;
+    opts.type = INDEX_PAGE;
 
     // read index page
     PageHandle page_handle;
diff --git a/be/src/olap/rowset/segment_v2/page_io.cpp b/be/src/olap/rowset/segment_v2/page_io.cpp
index 31a7ac2..f59174c 100644
--- a/be/src/olap/rowset/segment_v2/page_io.cpp
+++ b/be/src/olap/rowset/segment_v2/page_io.cpp
@@ -112,7 +112,7 @@ Status PageIO::read_and_decompress_page(const PageReadOptions& opts, PageHandle*
     auto cache = StoragePageCache::instance();
     PageCacheHandle cache_handle;
     StoragePageCache::CacheKey cache_key(opts.rblock->path(), opts.page_pointer.offset);
-    if (opts.use_page_cache && cache->lookup(cache_key, &cache_handle)) {
+    if (opts.use_page_cache && cache->is_cache_available(opts.type) && cache->lookup(cache_key, &cache_handle, opts.type)) {
         // we find page in cache, use it
         *handle = PageHandle(std::move(cache_handle));
         opts.stats->cached_pages_num++;
@@ -189,9 +189,9 @@ Status PageIO::read_and_decompress_page(const PageReadOptions& opts, PageHandle*
     }
 
     *body = Slice(page_slice.data, page_slice.size - 4 - footer_size);
-    if (opts.use_page_cache) {
+    if (opts.use_page_cache && cache->is_cache_available(opts.type)) {
         // insert this page into cache and return the cache handle
-        cache->insert(cache_key, page_slice, &cache_handle, opts.kept_in_memory);
+        cache->insert(cache_key, page_slice, &cache_handle, opts.type, opts.kept_in_memory);
         *handle = PageHandle(std::move(cache_handle));
     } else {
         *handle = PageHandle(page_slice);
diff --git a/be/src/olap/rowset/segment_v2/page_io.h b/be/src/olap/rowset/segment_v2/page_io.h
index e9cce2a..2cd79f3 100644
--- a/be/src/olap/rowset/segment_v2/page_io.h
+++ b/be/src/olap/rowset/segment_v2/page_io.h
@@ -54,6 +54,10 @@ struct PageReadOptions {
     // if true, use DURABLE CachePriority in page cache
     // currently used for in memory olap table
     bool kept_in_memory = false;
+    // for page cache allocation
+    // page types are divided into DATA_PAGE & INDEX_PAGE
+    // INDEX_PAGE including index_page, dict_page and short_key_page
+    PageTypePB type;
 
     void sanity_check() const {
         CHECK_NOTNULL(rblock);
diff --git a/be/src/olap/rowset/segment_v2/segment.cpp b/be/src/olap/rowset/segment_v2/segment.cpp
index 68ddb6d..db610a9 100644
--- a/be/src/olap/rowset/segment_v2/segment.cpp
+++ b/be/src/olap/rowset/segment_v2/segment.cpp
@@ -143,6 +143,7 @@ Status Segment::_load_index() {
         opts.codec = nullptr; // short key index page uses NO_COMPRESSION for now
         OlapReaderStatistics tmp_stats;
         opts.stats = &tmp_stats;
+        opts.type = INDEX_PAGE;
 
         Slice body;
         PageFooterPB footer;
diff --git a/be/src/runtime/exec_env_init.cpp b/be/src/runtime/exec_env_init.cpp
index 1f725b9..d3a8bf4 100644
--- a/be/src/runtime/exec_env_init.cpp
+++ b/be/src/runtime/exec_env_init.cpp
@@ -192,7 +192,8 @@ Status ExecEnv::_init_mem_tracker() {
         LOG(WARNING) << "Config storage_page_cache_limit is greater than memory size, config="
                      << config::storage_page_cache_limit << ", memory=" << MemInfo::physical_mem();
     }
-    StoragePageCache::create_global_cache(storage_cache_limit);
+    int32_t index_page_cache_percentage = config::index_page_cache_percentage;
+    StoragePageCache::create_global_cache(storage_cache_limit, index_page_cache_percentage);
 
     // TODO(zc): The current memory usage configuration is a bit confusing,
     // we need to sort out the use of memory
diff --git a/be/test/olap/page_cache_test.cpp b/be/test/olap/page_cache_test.cpp
index cdd12a3..0abd074 100644
--- a/be/test/olap/page_cache_test.cpp
+++ b/be/test/olap/page_cache_test.cpp
@@ -27,22 +27,25 @@ public:
     virtual ~StoragePageCacheTest() {}
 };
 
-TEST(StoragePageCacheTest, normal) {
-    StoragePageCache cache(kNumShards * 2048);
+// All cache space is allocated to data pages
+TEST(StoragePageCacheTest, data_page_only) {
+    StoragePageCache cache(kNumShards * 2048, 0);
 
     StoragePageCache::CacheKey key("abc", 0);
     StoragePageCache::CacheKey memory_key("mem", 0);
 
+    segment_v2::PageTypePB page_type = segment_v2::DATA_PAGE;
+
     {
         // insert normal page
         char* buf = new char[1024];
         PageCacheHandle handle;
         Slice data(buf, 1024);
-        cache.insert(key, data, &handle, false);
+        cache.insert(key, data, &handle, page_type, false);
 
         ASSERT_EQ(handle.data().data, buf);
 
-        auto found = cache.lookup(key, &handle);
+        auto found = cache.lookup(key, &handle, page_type);
         ASSERT_TRUE(found);
         ASSERT_EQ(buf, handle.data().data);
     }
@@ -52,11 +55,11 @@ TEST(StoragePageCacheTest, normal) {
         char* buf = new char[1024];
         PageCacheHandle handle;
         Slice data(buf, 1024);
-        cache.insert(memory_key, data, &handle, true);
+        cache.insert(memory_key, data, &handle, page_type, true);
 
         ASSERT_EQ(handle.data().data, buf);
 
-        auto found = cache.lookup(memory_key, &handle);
+        auto found = cache.lookup(memory_key, &handle, page_type);
         ASSERT_TRUE(found);
     }
 
@@ -65,23 +68,182 @@ TEST(StoragePageCacheTest, normal) {
         StoragePageCache::CacheKey key("bcd", i);
         PageCacheHandle handle;
         Slice data(new char[1024], 1024);
-        cache.insert(key, data, &handle, false);
+        cache.insert(key, data, &handle, page_type, false);
     }
 
     // cache miss
     {
         PageCacheHandle handle;
         StoragePageCache::CacheKey miss_key("abc", 1);
-        auto found = cache.lookup(miss_key, &handle);
+        auto found = cache.lookup(miss_key, &handle, page_type);
         ASSERT_FALSE(found);
     }
 
     // cache miss for eliminated key
     {
         PageCacheHandle handle;
-        auto found = cache.lookup(key, &handle);
+        auto found = cache.lookup(key, &handle, page_type);
         ASSERT_FALSE(found);
     }
+
+}
+
+// All cache space is allocated to index pages
+TEST(StoragePageCacheTest, index_page_only) {
+    StoragePageCache cache(kNumShards * 2048, 100);
+
+    StoragePageCache::CacheKey key("abc", 0);
+    StoragePageCache::CacheKey memory_key("mem", 0);
+
+    segment_v2::PageTypePB page_type = segment_v2::INDEX_PAGE;
+
+    {
+        // insert normal page
+        char* buf = new char[1024];
+        PageCacheHandle handle;
+        Slice data(buf, 1024);
+        cache.insert(key, data, &handle, page_type, false);
+
+        ASSERT_EQ(handle.data().data, buf);
+
+        auto found = cache.lookup(key, &handle, page_type);
+        ASSERT_TRUE(found);
+        ASSERT_EQ(buf, handle.data().data);
+    }
+
+    {
+        // insert in_memory page
+        char* buf = new char[1024];
+        PageCacheHandle handle;
+        Slice data(buf, 1024);
+        cache.insert(memory_key, data, &handle, page_type, true);
+
+        ASSERT_EQ(handle.data().data, buf);
+
+        auto found = cache.lookup(memory_key, &handle, page_type);
+        ASSERT_TRUE(found);
+    }
+
+    // put too many page to eliminate first page
+    for (int i = 0; i < 10 * kNumShards; ++i) {
+        StoragePageCache::CacheKey key("bcd", i);
+        PageCacheHandle handle;
+        Slice data(new char[1024], 1024);
+        cache.insert(key, data, &handle, page_type, false);
+    }
+
+    // cache miss
+    {
+        PageCacheHandle handle;
+        StoragePageCache::CacheKey miss_key("abc", 1);
+        auto found = cache.lookup(miss_key, &handle, page_type);
+        ASSERT_FALSE(found);
+    }
+
+    // cache miss for eliminated key
+    {
+        PageCacheHandle handle;
+        auto found = cache.lookup(key, &handle, page_type);
+        ASSERT_FALSE(found);
+    }
+
+}
+
+// Cache space is allocated by index_page_cache_ratio
+TEST(StoragePageCacheTest, mixed_pages) {
+    StoragePageCache cache(kNumShards * 2048, 10);
+
+    StoragePageCache::CacheKey data_key("data", 0);
+    StoragePageCache::CacheKey index_key("index", 0);
+    StoragePageCache::CacheKey data_key_mem("data_mem", 0);
+    StoragePageCache::CacheKey index_key_mem("index_mem", 0);
+
+    segment_v2::PageTypePB page_type_data = segment_v2::DATA_PAGE;
+    segment_v2::PageTypePB page_type_index = segment_v2::INDEX_PAGE;
+
+    {
+        // insert both normal pages
+        char* buf_data = new char[1024];
+        char* buf_index = new char[1024];
+        PageCacheHandle data_handle, index_handle;
+        Slice data(buf_data, 1024), index(buf_index, 1024);
+        cache.insert(data_key, data, &data_handle, page_type_data, false);
+        cache.insert(index_key, index, &index_handle, page_type_index, false);
+
+        ASSERT_EQ(data_handle.data().data, buf_data);
+        ASSERT_EQ(index_handle.data().data, buf_index);
+
+        auto found_data = cache.lookup(data_key, &data_handle, page_type_data);
+        auto found_index = cache.lookup(index_key, &index_handle, page_type_index);
+        ASSERT_TRUE(found_data);
+        ASSERT_TRUE(found_index);
+        ASSERT_EQ(buf_data, data_handle.data().data);
+        ASSERT_EQ(buf_index, index_handle.data().data);
+    }
+
+    {
+        // insert both in_memory pages
+        char* buf_data = new char[1024];
+        char* buf_index = new char[1024];
+        PageCacheHandle data_handle, index_handle;
+        Slice data(buf_data, 1024), index(buf_index, 1024);
+        cache.insert(data_key_mem, data, &data_handle, page_type_data, true);
+        cache.insert(index_key_mem, index, &index_handle, page_type_index, true);
+
+        ASSERT_EQ(data_handle.data().data, buf_data);
+        ASSERT_EQ(index_handle.data().data, buf_index);        
+
+        auto found_data = cache.lookup(data_key_mem, &data_handle, page_type_data);
+        auto found_index = cache.lookup(index_key_mem, &index_handle, page_type_index);
+        ASSERT_TRUE(found_data);
+        ASSERT_TRUE(found_index);
+    }
+
+    // put too many page to eliminate first page of both cache
+    for (int i = 0; i < 10 * kNumShards; ++i) {
+        StoragePageCache::CacheKey key("bcd", i);
+        PageCacheHandle handle;
+        Slice data(new char[1024], 1024), index(new char[1024], 1024);
+        cache.insert(key, data, &handle, page_type_data, false);
+        cache.insert(key, index, &handle, page_type_index, false);
+    }
+
+    // cache miss by key
+    {
+        PageCacheHandle data_handle, index_handle;
+        StoragePageCache::CacheKey miss_key("abc", 1);
+        auto found_data = cache.lookup(miss_key, &data_handle, page_type_data);
+        auto found_index = cache.lookup(miss_key, &index_handle, page_type_index);
+        ASSERT_FALSE(found_data);
+        ASSERT_FALSE(found_index);
+    }
+
+    // cache miss by page type
+    {
+        PageCacheHandle data_handle, index_handle;
+        StoragePageCache::CacheKey miss_key_data("data_miss", 1);
+        StoragePageCache::CacheKey miss_key_index("index_miss", 1);
+        char* buf_data = new char[1024];
+        char* buf_index = new char[1024];
+        Slice data(buf_data, 1024), index(buf_index, 1024);
+        cache.insert(miss_key_data, data, &data_handle, page_type_data, false);
+        cache.insert(miss_key_index, index, &index_handle, page_type_index, false);
+        
+        auto found_data = cache.lookup(miss_key_data, &data_handle, page_type_index);
+        auto found_index = cache.lookup(miss_key_index, &index_handle, page_type_data);
+        ASSERT_FALSE(found_data);
+        ASSERT_FALSE(found_index);
+    }
+
+    // cache miss for eliminated key
+    {
+        PageCacheHandle data_handle, index_handle;
+        auto found_data = cache.lookup(data_key, &data_handle, page_type_data);
+        auto found_index = cache.lookup(index_key, &index_handle, page_type_index);
+        ASSERT_FALSE(found_data);
+        ASSERT_FALSE(found_index);
+    }
+
 }
 
 } // namespace doris
diff --git a/be/test/olap/rowset/beta_rowset_test.cpp b/be/test/olap/rowset/beta_rowset_test.cpp
index ec864b9..6191e6f 100644
--- a/be/test/olap/rowset/beta_rowset_test.cpp
+++ b/be/test/olap/rowset/beta_rowset_test.cpp
@@ -353,7 +353,7 @@ TEST_F(BetaRowsetTest, BasicFunctionTest) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/rowset_converter_test.cpp b/be/test/olap/rowset/rowset_converter_test.cpp
index 47c8e69..867c711 100644
--- a/be/test/olap/rowset/rowset_converter_test.cpp
+++ b/be/test/olap/rowset/rowset_converter_test.cpp
@@ -298,7 +298,7 @@ TEST_F(RowsetConverterTest, TestConvertBetaRowsetToAlpha) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/bitmap_index_test.cpp b/be/test/olap/rowset/segment_v2/bitmap_index_test.cpp
index a04c21e..8603e4c 100644
--- a/be/test/olap/rowset/segment_v2/bitmap_index_test.cpp
+++ b/be/test/olap/rowset/segment_v2/bitmap_index_test.cpp
@@ -238,7 +238,7 @@ TEST_F(BitmapIndexTest, test_null) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/bloom_filter_index_reader_writer_test.cpp b/be/test/olap/rowset/segment_v2/bloom_filter_index_reader_writer_test.cpp
index 8795699..ee82e0d 100644
--- a/be/test/olap/rowset/segment_v2/bloom_filter_index_reader_writer_test.cpp
+++ b/be/test/olap/rowset/segment_v2/bloom_filter_index_reader_writer_test.cpp
@@ -291,7 +291,7 @@ TEST_F(BloomFilterIndexReaderWriterTest, test_decimal) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/column_reader_writer_test.cpp b/be/test/olap/rowset/segment_v2/column_reader_writer_test.cpp
index 057ccfb..bf2e7dad 100644
--- a/be/test/olap/rowset/segment_v2/column_reader_writer_test.cpp
+++ b/be/test/olap/rowset/segment_v2/column_reader_writer_test.cpp
@@ -673,7 +673,7 @@ TEST_F(ColumnReaderWriterTest, test_default_value) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/ordinal_page_index_test.cpp b/be/test/olap/rowset/segment_v2/ordinal_page_index_test.cpp
index caa3ab7..9c4fd85 100644
--- a/be/test/olap/rowset/segment_v2/ordinal_page_index_test.cpp
+++ b/be/test/olap/rowset/segment_v2/ordinal_page_index_test.cpp
@@ -158,7 +158,7 @@ TEST_F(OrdinalPageIndexTest, one_data_page) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/segment_test.cpp b/be/test/olap/rowset/segment_v2/segment_test.cpp
index f4971b6..34291ea 100644
--- a/be/test/olap/rowset/segment_v2/segment_test.cpp
+++ b/be/test/olap/rowset/segment_v2/segment_test.cpp
@@ -1160,7 +1160,7 @@ TEST_F(SegmentReaderWriterTest, TestBloomFilterIndexUniqueModel) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     ::testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/be/test/olap/rowset/segment_v2/zone_map_index_test.cpp b/be/test/olap/rowset/segment_v2/zone_map_index_test.cpp
index f2f49fc..e795fb5 100644
--- a/be/test/olap/rowset/segment_v2/zone_map_index_test.cpp
+++ b/be/test/olap/rowset/segment_v2/zone_map_index_test.cpp
@@ -175,7 +175,7 @@ TEST_F(ColumnZoneMapTest, NormalTestCharPage) {
 } // namespace doris
 
 int main(int argc, char** argv) {
-    doris::StoragePageCache::create_global_cache(1 << 30);
+    doris::StoragePageCache::create_global_cache(1 << 30, 0.1);
     testing::InitGoogleTest(&argc, argv);
     return RUN_ALL_TESTS();
 }
diff --git a/docs/en/administrator-guide/config/be_config.md b/docs/en/administrator-guide/config/be_config.md
index 2a9096f..79e8813 100644
--- a/docs/en/administrator-guide/config/be_config.md
+++ b/docs/en/administrator-guide/config/be_config.md
@@ -720,6 +720,11 @@ Indicates how many tablets in this data directory failed to load. At the same ti
 
 ### `storage_page_cache_limit`
 
+### `index_page_cache_percentage`
+* Type: int32
+* Description: Index page cache as a percentage of total storage page cache, value range is [0, 100]
+* Default value: 10
+
 ### `storage_root_path`
 
 * Type: string
diff --git a/docs/zh-CN/administrator-guide/config/be_config.md b/docs/zh-CN/administrator-guide/config/be_config.md
index 0bc0d38..3f6345b 100644
--- a/docs/zh-CN/administrator-guide/config/be_config.md
+++ b/docs/zh-CN/administrator-guide/config/be_config.md
@@ -720,6 +720,11 @@ load tablets from header failed, failed tablets size: xxx, path=xxx
 
 ### `storage_page_cache_limit`
 
+### `index_page_cache_percentage`
+* 类型:int32
+* 描述:索引页缓存占总页面缓存的百分比,取值为[0, 100]。
+* 默认值:10
+
 ### `storage_root_path`
 
 * 类型:string


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org