Flink RocksDB 内存管理

RocksDB 的内存管理在 Flink 1.10 版本之前是不受管控的,社区利用 RocksDB 已有的一些内存控制上的优化,在去年对 Flink 中 RocksDB 的使用做了一系列的改造(详见 FLINK-7289)。

注:阅读本文需要对 Flink RocksDBStateBackend 机制有一定了解。

RocksDB 内存使用

RocksDB 使用的内存可以分为以下若干部分(详见官方文档):

  • Memtable(Write Buffer): 数据写入时会先写入到 Memtable 中,满足一定条件后再 flush 到磁盘上
  • Block Cache: 读取数据时的内存 Cache,以 Block 为单位进行存储
  • Indexes and filters: 访问 Block 时使用的 Index 和 Filter,用来判断读取的 Key 对应的位置
  • Blocks pinned by iterators:遍历数据时,处于性能考虑,会在 RocksDB 中的各个 Level 初始化若干个 Block,进行批量初始化和数据遍历(而不是一条条从磁盘读取数据)

上述内存都是使用 Native Memory,在 Flink 中通常占用比较大的内存会是 Memtable、BlockCache 和 Indexes 和 BloomFilters。需要注意的是,上述内存在 Flink-1.10 版本之前只能做 ColumnFamily 级别的配置,也就是每一个 State 都对应着一组全套的内存消耗,所以定义的 State 越多,使用的内存会线性增长直至 OOM。

RocksDB Memory in Flink

RocksDB 通常是用于线上服务的数据存储,部署模式上通常会以大内存(几十 GB)的实例存在,而在 Apache Flink 这样的分布式计算应用中,每个实例的资源相对较小,对于内存的控制需要更加严格,否则稍有不慎就会导致 Container OOM 退出。

Flink-1.9 中比较简单粗暴,用户可控的内存参数如下:

参数 含义 默认值
state.backend.rocksdb.block.cache-size 每个 ColumnFamily 使用的 Block Cache 大小 8MB
state.backend.rocksdb.writebuffer.size 每个 ColumnFamily 使用的 Memtable 大小 64MB
state.backend.rocksdb.writebuffer.count 每个 ColumnFamily 使用的 Memtable 个数 2

对比上面的 RocksDB 内存使用可以看到,Flink-1.9 提供的内存参数较为简单,只能控制每个 ColumnFamily 使用的 Native Memory 中的 Memtable 和 Block Cache,对于其他部分的内存使用,以及 RocksDB 整体的内存使用,都是没有任何办法控制的。

相比 1.9 版本中松散的内存管理方式,1.11 版本将进程内 RocksDB 使用的内存通过 WriteBufferManagerBlock Cache 将同一进程内 RocksDB 使用的内存给统一起来。 这个统一用到了 RocksDB 的几个特性:

  • WriteBufferManager 可以使用 Cache 作为内存管理容器,这样就通过 Cache 将 Write Buffer 和 Block Cache 的内存使用都控制起来
  • Index 和 Filter 可以放入 Block Cache 中,并使用 Partitioned Index Filters 特性保证 Index 和 Filter 不被 data cache 给 evict 掉(保证了读取 index 和 filter 的性能)

使用内存控制的前提是用户开启了 Managed Memory 或者设置了 state.backend.rocksdb.memory.fixed-per-slot,根据用户给出的 RocksDB 可使用的内存,基于给定的内存,对 Cache 的大小进行计算,参数如下:

参数 含义 默认值
state.backend.rocksdb.memory.write-buffer-ratio WriteBuffer 内存占用比例 0.5
state.backend.rocksdb.memory.high-prio-pool-ratio Cache 中 index 和 filter 内存占用比例 0.1

细心的同学可能会发现 Blocks pinned by iterators 这部分占用的内存并没有在配置中出现,实际上 Flink 也并没有对这部分内存进行专项的配置,并且由于 RocksDB 的某些 bug,Cache 不能开启 “strict capacity limit” 模式,所以在 Flink 中做了一系列的计算,根据用户指定的参数,实际配置的内存会相对较小,通过留有一定的 buffer 来保证内存是够用的。相关计算过程如下(摘自 RocksDBMemoryControllerUtils.java#L64):

    /**
     * Calculate the actual memory capacity of cache, which would be shared among rocksDB instance(s).
     * We introduce this method because:
     * a) We cannot create a strict capacity limit cache util FLINK-15532 resolved.
     * b) Regardless of the memory usage of blocks pinned by RocksDB iterators,
     * which is difficult to calculate and only happened when we iterator entries in RocksDBMapState, the overuse of memory is mainly occupied by at most half of the write buffer usage.
     * (see <a href="https://github.com/dataArtisans/frocksdb/blob/958f191d3f7276ae59b270f9db8390034d549ee0/include/rocksdb/write_buffer_manager.h#L51">the flush implementation of write buffer manager</a>).
     * Thus, we have four equations below:
     *   write_buffer_manager_memory = 1.5 * write_buffer_manager_capacity
     *   write_buffer_manager_memory = total_memory_size * write_buffer_ratio
     *   write_buffer_manager_memory + other_part = total_memory_size
     *   write_buffer_manager_capacity + other_part = cache_capacity
     * And we would deduce the formula:
     *   cache_capacity =  (3 - write_buffer_ratio) * total_memory_size / 3
     *   write_buffer_manager_capacity = 2 * total_memory_size * write_buffer_ratio / 3
     */

新版本中存在的问题

新版本对 RocksDB 确实有了更好的控制,但实际上,我们在内部使用中关闭了 Managed Memory,也就不会使用上面的一整套对应的内存管理机制。(更多是从用户角度考虑

(1) 内存使用虚高,用户盲目扩大内存

Flink 1.9 中,虽然无法精确控制内存大小,但内存的使用是真实反映在监控上,比如内存配多了,还是配少了,从监控上可以非常清晰地看出来。而在 Flink 1.11 中,Managed Memory 中的内存中大部分都会在启动时就被占用,并且内存配置的大小是否合适,并没有直观的监控可以看到(比如 write-buffer-ratio 应该如何调整等)。

从用户的角度来看,在总内存不变的情况下,之前内存使用率是 50%,现在变成 80% 了,而实际作业又不需要这么多的内存。大部分用户看到堆外内存快占满时,第一选择都是直接提高内存,然后看到的现象仍然是堆外内存大部分被占满。

(2) 内存管理做不到完全精确,对于绝大部分作业 OOM 的概率和次数提高。

这里主要是指在使用 Managed Memory 过程中,遇到了很多的问题,包括:

  • FLINK-24120:每次产生一个新的 savepoint,作业的堆外内存就会增加一小部分,最后查到是 glibc 下 ARENA 泄漏的问题
  • RocksDB 中的 Block Cache 仍无法使用精确的内存控制(详见FLINK-15532),对用户仍然无法做一个确定性的保证

引用