简介
ClickHouse 是最快速、资源效率最高的开源数据库,适用于实时应用程序和分析。作为其组件之一,ClickHouse Keeper 是 ZooKeeper 的快速、资源效率更高且功能更丰富的替代方案。这个开源组件提供高度可靠的元数据存储,以及协调和同步机制。它最初是为与 ClickHouse 一起使用而开发的,当 ClickHouse 以分布式系统部署在自托管设置或托管产品(如 CloudHouse Cloud)中时。但是,我们相信更广泛的社区可以在其他用例中从此项目中受益。
在这篇文章中,我们描述了 ClickHouse Keeper 的动机、优势和开发,并预览了我们接下来计划的改进。此外,我们还介绍了一个可重用的基准测试套件,它使我们能够轻松地模拟和基准测试典型的 ClickHouse Keeper 使用模式。基于此,我们展示了基准测试结果,突出显示 ClickHouse Keeper 在数据量相同的情况下,内存使用量比 ZooKeeper 少 46 倍,同时保持了接近 ZooKeeper 的性能。
动机
现代分布式系统需要共享且可靠的信息存储库和共识系统,用于协调和同步分布式操作。对于 ClickHouse,最初选择ZooKeeper来实现这一点。它通过广泛的使用而变得可靠,提供了简单而强大的 API,并提供了合理的性能。
然而,不仅是性能,资源效率和可扩展性一直是 ClickHouse 的首要优先事项。ZooKeeper 作为一个 Java 生态系统项目,与我们主要的 C++ 代码库并不十分契合,并且随着我们在越来越高的规模上使用它,我们开始遇到资源使用和操作挑战。为了克服 ZooKeeper 的这些缺点,我们从头开始构建了 ClickHouse Keeper,同时考虑了我们的项目需要解决的其他要求和目标。
ClickHouse Keeper 是 ZooKeeper 的即插即用替代品,具有完全兼容的客户端协议和相同的数据模型。除此之外,它还提供以下好处
- 更轻松的设置和操作:ClickHouse Keeper 是用 C++ 而不是 Java 实现的,因此可以嵌入在 ClickHouse 中运行或独立运行
- 由于更好的压缩,快照和日志消耗的磁盘空间更少
- 对默认数据包和节点数据大小没有限制(在 ZooKeeper 中是 1 MB)
- 没有ZXID 溢出问题(在 ZooKeeper 中,每 2B 事务会强制重启)
- 由于使用了更好的分布式共识协议,网络分区后恢复速度更快
- 额外的一致性保证:ClickHouse Keeper 提供与 ZooKeeper 相同的一致性保证 - 线性化写入加上同一会话内操作的严格排序。此外,可选地(通过quorum_reads 设置),ClickHouse Keeper 提供线性化读取。
- ClickHouse Keeper 资源效率更高,并且对于相同的数据量使用更少的内存(我们将在本博客稍后部分进行演示)
ClickHouse Keeper 的开发开始于 2021 年 2 月,作为 ClickHouse 服务器中的嵌入式服务。同年,引入了独立模式,并Jepsen 测试被添加 - 每 6 小时,我们运行自动化的测试,其中包含几种不同的工作流程和故障场景,以验证共识机制的正确性。
在撰写本博客时,ClickHouse Keeper 已经为生产环境做好准备超过一年半,并且自 2022 年 5 月首次私人预览发布以来,已在我们自己的ClickHouse Cloud 中大规模部署。
在本博客的其余部分中,我们有时将 ClickHouse Keeper 简称为“Keeper”,正如我们内部经常称呼它一样。
在 ClickHouse 中的使用
通常,任何需要多个 ClickHouse 服务器之间保持一致性的操作都依赖于 Keeper
- Keeper 为自托管无共享 ClickHouse 集群中的数据复制提供协调系统
- 对于 mergetree 引擎系列的复制表,自动插入重复数据删除是基于存储在 Keeper 中的块哈希和实现的
- Keeper 为 part 名称(基于顺序块编号)和将 part 合并和mutation 分配给特定集群节点提供共识
- Keeper 在 KeeperMap 表引擎的底层使用,该引擎允许您将 Keeper 用作具有线性化写入和顺序一致性读取的一致性键值存储
- 阅读关于一个应用程序利用它在 ClickHouse 之上实现任务调度队列的信息
- Kafka Connect Sink 使用此表引擎作为可靠的状态存储,用于实现仅一次交付保证
- Keeper 跟踪 S3Queue 表引擎中已消耗的文件
- 复制数据库引擎将所有元数据存储在 Keeper 中
- Keeper 用于协调使用 ON CLUSTER 子句的备份
- 用户定义的函数可以存储在 Keeper 中
- 访问控制信息可以存储在 Keeper 中
- Keeper 在 ClickHouse Cloud 中用作所有元数据的共享中心存储
观察 Keeper
在以下部分中,为了观察(并在基准测试中稍后建模)ClickHouse Cloud 与 Keeper 的一些交互,我们将来自 WikiStat 数据集的一个月的数据加载到 表中,该表位于具有 3 个节点的 ClickHouse Cloud 服务中。每个节点有 30 个 CPU 核心和 120 GB RAM。每个服务都使用自己的专用 ClickHouse Keeper 服务,该服务由 3 台服务器组成,每台 Keeper 服务器有 3 个 CPU 核心和 2 GB RAM。
下图说明了此数据加载场景:
① 数据加载
通过数据加载 查询,我们使用所有三台 ClickHouse 服务器在约 100 秒内并行加载了来自约 740 个压缩文件(一个文件代表特定日期的特定小时)的约 46.4 亿行数据。单个 ClickHouse 服务器上的峰值主内存使用量约为 107 GB
0 rows in set. Elapsed: 101.208 sec. Processed 4.64 billion rows, 40.58 GB (45.86 million rows/s., 400.93 MB/s.)
Peak memory usage: 107.75 GiB.
② Part 创建
为了存储数据,3 台 ClickHouse 服务器一起创建了 240 个初始 part 在对象存储中。每个初始 part 的平均行数约为 1900 万行。平均大小约为 100 MiB,插入的总行数为 46.4 亿
┌─parts──┬─rows_avg──────┬─size_avg───┬─rows_total───┐
│ 240.00 │ 19.34 million │ 108.89 MiB │ 4.64 billion │
└────────┴───────────────┴────────────┴──────────────┘
由于我们的数据加载查询使用了 s3Cluster 表函数,因此初始 part 的创建均匀分布在我们的 ClickHouse Cloud 服务的 3 台 ClickHouse 服务器上
┌─n─┬─parts─┬─rows_total───┐
│ 1 │ 86.00 │ 1.61 billion │
│ 2 │ 76.00 │ 1.52 billion │
│ 3 │ 78.00 │ 1.51 billion │
└───┴───────┴──────────────┘
③ Part 合并
在数据加载期间,在后台,ClickHouse 执行了 1706 次 part 合并
┌─merges─┐
│ 1706 │
└────────┘
④ Keeper 交互
ClickHouse Cloud 完全分离了数据存储和元数据存储与服务器。所有数据 part 都存储在共享对象存储中,所有元数据都存储在 Keeper 中。当 ClickHouse 服务器已将新 part 写入对象存储(请参阅上面的 ②)或将某些 part 合并为新的更大的 part(请参阅上面的 ③)时,则此 ClickHouse 服务器使用 multi-write 事务请求来更新 Keeper 中有关新 part 的元数据。此信息包括 part 的名称、属于 part 的文件以及与文件对应的 blob 在对象存储中的位置。每个服务器都有一个本地缓存,其中包含元数据的子集,并通过 Keeper 实例通过基于 watch 的订阅机制自动获取有关数据更改的通知。
对于我们前面提到的初始 part 创建和后台 part 合并,总共执行了约 1.8 万个 Keeper 请求。这包括约 1.2 万个 multi-write 事务请求(仅包含 write-subrequests)。所有其他请求都是读和写请求的混合。此外,ClickHouse 服务器从 Keeper 收到了约 800 个 watch 通知
total_requests: 17705
multi_requests: 11642
watch_notifications: 822
我们可以看到这些请求是如何发送的,以及 watch 通知是如何从所有三个 ClickHouse 节点相当均匀地接收的
┌─n─┬─total_requests─┬─multi_requests─┬─watch_notifications─┐
│ 1 │ 5741 │ 3671 │ 278 │
│ 2 │ 5593 │ 3685 │ 269 │
│ 3 │ 6371 │ 4286 │ 275 │
└───┴────────────────┴────────────────┴─────────────────────┘
以下两张图表可视化了数据加载过程中的 Keeper 请求: 我们可以看到,约 70% 的 Keeper 请求是 multi-write 事务。
请注意,Keeper 请求的数量可能会因 ClickHouse 集群大小、摄取设置和数据大小而异。我们简要演示了这三个因素如何影响生成的 Keeper 请求的数量。
ClickHouse 集群大小
如果我们使用 10 个而不是 3 个服务器并行加载数据,我们摄取数据的速度会提高 3 倍以上(使用 SharedMergeTree)
0 rows in set. Elapsed: 33.634 sec. Processed 4.64 billion rows, 40.58 GB (138.01 million rows/s., 1.21 GB/s.)
Peak memory usage: 57.09 GiB.
服务器数量越多,生成的 Keeper 请求量也超过 3 倍
total_requests: 60925
multi_requests: 41767
watch_notifications: 3468
摄取设置
对于我们使用 3 个 ClickHouse 服务器运行的原始数据加载,我们配置了每个初始 part 的最大大小约为 2500 万行,以加快摄取速度,但代价是更高的内存使用率。相反,如果我们使用每个初始 part 约 100 万行的默认值运行相同的数据加载,则数据加载速度较慢,但每个 ClickHouse 服务器使用的主内存减少约 9 倍
0 rows in set. Elapsed: 121.421 sec. Processed 4.64 billion rows, 40.58 GB (38.23 million rows/s., 334.19 MB/s.)
Peak memory usage: 12.02 GiB.
并且创建了约 4 千个而不是 240 个初始 part
┌─parts─────────┬─rows_avg─────┬─size_avg─┬─rows_total───┐
│ 4.24 thousand │ 1.09 million │ 9.20 MiB │ 4.64 billion │
└───────────────┴──────────────┴──────────┴──────────────┘
这导致更多的 part 合并
┌─merges─┐
│ 9094 │
└────────┘
并且我们获得了更高数量的 Keeper 请求(约 14.7 万个而不是约 1.7 万个)
total_requests: 147540
multi_requests: 105951
watch_notifications: 7439
数据大小
类似地,如果我们加载更多数据(每个初始 part 的默认值为约 100 万行),例如来自 WikiStat 数据集的六个月的数据,那么我们的服务将获得约 2.4 万个初始 part
┌─parts──────────┬─rows_avg─────┬─size_avg─┬─rows_total────┐
│ 23.75 thousand │ 1.10 million │ 9.24 MiB │ 26.23 billion │
└────────────────┴──────────────┴──────────┴───────────────┘
这导致更多的合并
┌─merges─┐
│ 28959 │
└────────┘
导致约 68 万个 Keeper 请求
total_requests: 680996
multi_requests: 474093
watch_notifications: 32779
基准测试 Keeper
我们开发了一个基准测试套件,名为 keeper-bench-suite,用于基准测试上面探讨的典型的 ClickHouse 与 Keeper 的交互。为此,keeper-bench-suite 允许模拟来自由 N
(例如 3)台服务器组成的 ClickHouse 集群的并行 Keeper 工作负载: 我们利用 keeper-bench,这是一个用于基准测试 Keeper 或任何 ZooKeeper 兼容系统的工具。借助该构建块,我们可以模拟和基准测试来自
N
台 ClickHouse 服务器的典型并行 Keeper 流量。此图显示了 Keeper Bench Suite 的完整架构,它使我们能够轻松设置和基准测试任意 Keeper 工作负载场景: 我们使用 AWS EC2 实例作为基准测试服务器,用于执行 Python 脚本,该脚本
① 通过启动 3 个合适的(例如,m6a.4xlarge)EC2 实例来设置和启动 3 节点 Keeper 集群,每个实例运行一个 Keeper docker 容器和两个包含 cAdvisor 和 Redis(cAdvisor 需要)的容器,用于监控本地 Keeper 容器的资源使用情况
② 使用预配置的工作负载配置启动 keeper-bench
③ 每 1 秒抓取每个 cAdvisor 和 Keeper 的 Prometheus 端点
④ 将抓取的指标(包括时间戳)写入 ClickHouse Cloud 服务中的两个表,这是通过 SQL 查询方便地分析指标和 Grafana 仪表板的基础
请注意,ClickHouse Keeper 和 ZooKeeper 都直接提供 Prometheus 端点。目前,这些端点只有很小的重叠,并且通常给出非常不同的指标,这使得很难比较它们,尤其是在内存和 CPU 使用率方面。因此,我们选择了基于 cAdvisor 的附加基本容器指标。此外,在 docker 容器中运行 Keeper 使我们能够轻松更改提供给 Keeper 的 CPU 核心数量和 RAM 大小。
配置参数
Keeper 的大小
我们使用 ClickHouse Keeper 和 ZooKeeper 的不同 docker 容器大小运行基准测试。例如,1 个 CPU 核心 + 1 GB RAM,3 个 CPU 核心 + 1 GB RAM,6 个 CPU 核心 + 6 GB RAM。
客户端和请求的数量
对于每个 Keeper 大小,我们模拟(使用 keeper-bench 的 concurrency 设置)不同数量的客户端(例如,ClickHouse 服务器)并行向 Keeper 发送请求:例如 3、10、100、500、1000。
从这些模拟客户端中的每一个,为了模拟短期和长期运行的 Keeper 会话,我们(使用 keeper-bench 的 iterations 设置)向 Keeper 发送总数在 1 万到约 1 千万之间的请求。这旨在测试任一组件的内存使用情况是否随时间变化。
工作负载
我们模拟了典型的 ClickHouse 工作负载,其中包含约 1/3 的写入和删除操作以及约 2/3 的读取操作。这反映了摄取、合并然后查询某些数据的场景。可以轻松定义和基准测试其他工作负载。
测量的指标
Prometheus 端点
我们使用 cAdvisor 的 Prometheus 端点来测量
- 主内存使用量 (container_memory_working_set_bytes)
- CPU 使用率 (container_cpu_usage_seconds_total)
我们使用 ClickHouse Keeper 和 ZooKeeper 的 Prometheus 端点来测量其他(所有可用)Keeper Prometheus 端点指标值。例如,对于 ZooKeeper,许多 JVM 特定指标(堆大小和使用率、垃圾回收等)。
运行时
我们还根据每次运行的最小和最大时间戳来测量 Keeper 处理所有请求的运行时。
结果
我们使用 keeper-bench-suite 来比较 ClickHouse Keeper 和 ZooKeeper 在我们的工作负载下的资源消耗和运行时。我们运行了每个基准测试配置 10 次,并将结果存储在 ClickHouse Cloud 服务中的两个表中。我们使用 SQL 查询生成了三个表格结果表
这些结果的列在此处描述。
我们对所有运行使用了 ClickHouse Keeper 23.5
和 ZooKeeper 3.8.
(捆绑了 OpenJDK 11
)。请注意,我们在此处不打印三个表格结果,因为每个表包含 216 行。您可以按照上面的链接检查结果。
示例结果
在这里,我们展示了两张图表,我们在图表中筛选了第 99 个百分位数结果,其中 Keeper 的两个版本都以 3 个 CPU 核心和 2 GB RAM 运行,处理来自 3 个模拟客户端(ClickHouse 服务器)并行发送的相同请求大小。这些可视化的表格结果在此处。
内存使用量
我们可以看到,对于我们模拟的工作负载,对于相同数量的处理请求,ClickHouse Keeper 始终比 ZooKeeper 使用更少的主内存。例如,对于基准测试运行 ③,处理由 3 个模拟 ClickHouse 服务器并行发送的 640 万个请求,ClickHouse Keeper 在运行 ④ 中使用的主内存比 ZooKeeper 少约 46 倍。
对于 ZooKeeper,我们对所有主要运行(①、②、③)使用了 1GiB JVM 堆大小配置(JVMFLAGS: -Xmx1024m -Xms1024m
),这意味着对于这些运行,提交的 JVM 内存(保留堆和非堆内存保证可供 Java 虚拟机使用)大小约为 1GiB(请参阅上图中的透明灰色条,了解使用了多少)。除了 docker 容器内存使用量(蓝色条)之外,我们还测量了在提交的 JVM 内存中实际使用的(堆和非堆)JVM 内存量(粉色条)。运行 JVM 本身存在一些轻微的容器内存开销(蓝色条和粉色条之间的差异)。但是,实际使用的 JVM 内存仍然始终显着大于 ClickHouse Keeper 的总体容器内存使用量。
此外,我们可以看到 ZooKeeper 在运行 ③ 中使用了完整的 1 GiB JVM 堆大小。我们对 ZooKeeper 进行了额外的运行 ④,JVM 堆大小增加到 2 GiB,导致 ZooKeeper 使用了其 2 GiB JVM 堆中的 1.56 GiB,运行时得到了改进,与 ClickHouse Keeper 的运行 ③ 的运行时相匹配。我们在下一张图中展示了上述所有运行的运行时。
我们可以在表格结果中看到,在 ZooKeeper 运行期间发生过几次(主要)垃圾回收。
运行时和 CPU 使用率
下图可视化了上一张图中讨论的运行的运行时和 CPU 使用率(带圈的数字在两张图中对齐): ClickHouse Keeper 的运行时与 ZooKeeper 的运行时非常接近。尽管使用的主内存(参见上一张图)和 CPU 明显更少。
扩展 Keeper
我们观察到 ClickHouse Cloud 在与 Keeper 交互时经常使用 multi-write 事务。我们更深入地研究 ClickHouse Cloud 与 Keeper 的交互,以草绘 ClickHouse 服务器使用的此类 Keeper 事务的两个主要场景。
自动插入重复数据删除
在上述场景中,server-2 ① 以 块-式 处理插入到表中的数据。对于当前块,server-2 ② 将数据写入对象存储中的新数据 part,并且 ③ 使用 Keeper 多写 事务 来存储关于新 part 的元数据在 Keeper 中,例如,与 part 文件对应的 blob 在对象存储中的位置。在存储此元数据之前,事务首先尝试将步骤 ① 中处理的块的哈希和存储在 Keeper 中的
去重日志
znode 中。如果相同的哈希和值 已经 存在于去重日志中,则整个事务 失败 (回滚)。此外,步骤 ② 中的数据 part 也被删除,因为该 part 中包含的数据过去已被插入。这种自动插入 去重 使 ClickHouse 插入具有 幂等性,因此具有容错性,允许客户端重试插入而不会冒数据重复的风险。成功后,事务触发子 watches,并且 ④ 所有 订阅 了 part-metadata znodes 事件的 ClickHouse 服务器都会被 Keeper 自动通知新的条目。这导致它们从 Keeper 将元数据更新提取到其本地元数据缓存中。
为服务器分配 part 合并
当 server-2 决定将一些 part 合并成一个更大的 part 时,服务器 ① 使用 Keeper 事务来标记要合并的 part 为锁定状态(以防止其他服务器合并它们)。接下来,server-2 ② 将这些 part 合并成一个新的更大的 part,并且 ③ 使用另一个 Keeper 事务来存储关于新 part 的元数据,这会触发 watches ④ 通知所有其他服务器关于新的元数据条目。
请注意,只有当 Keeper 原子地且顺序地执行此类 Keeper 事务时,上述场景才能正确工作。否则,两个 ClickHouse 服务器同时并行发送相同的数据可能会导致两者都在去重日志中找不到数据的哈希和,从而导致对象存储中的数据重复。或者多台服务器会合并相同的 part。为了防止这种情况,ClickHouse 服务器依赖于 Keeper 的全有或全无多写事务及其线性化写入保证。
线性一致性 vs 多核处理
ZooKeeper 和 ClickHouse Keeper 中的 共识算法,分别是 ZAB 和 Raft,都确保多个分布式服务器可以可靠地就相同的信息达成一致。例如,在上面的例子中,哪些 part 允许被合并。
ZAB 是 ZooKeeper 的专用共识机制,并且至少自 2008 年以来一直在开发 至今。
我们选择 Raft 作为我们的共识机制,因为它的算法简单且 易于理解,并且当我们于 2021 年启动 Keeper 项目时,可以获得轻量级且易于集成的 C++ 库。
然而,所有共识算法都是彼此同构的。对于 线性一致性 写入,(依赖的)转换和事务内的写入操作必须严格按顺序一次处理一个,无论使用哪种共识算法。假设 ClickHouse 服务器并行向 Keeper 发送事务,并且这些事务是相关的,因为它们写入相同的 znodes(例如,在本节开始的示例场景中的 去重日志
)。在这种情况下,Keeper 只能通过严格顺序地执行此类事务及其操作来保证和实现线性一致性: 为此,ZooKeeper 使用单线程 请求处理器 实现写入请求处理,而 Keeper 的 NuRaft 实现使用单线程 全局队列。
通常,线性一致性使得难以垂直(更多 CPU 核心)或水平(更多服务器)扩展写入处理速度。可以分析和识别独立的事务并并行运行它们,但目前 ZooKeeper 和 ClickHouse Keeper 都没有实现这一点。此图表(我们在此处过滤了第 99 百分位结果)突出了这一点: ZooKeeper 和 ClickHouse Keeper 都在 1、3 和 6 个 CPU 核心上运行,并处理从 500 个客户端并行发送的 128 万个总请求。
(非线性一致性)读取请求和辅助任务(管理网络请求、批量数据等)的性能在理论上可以通过 ZAB 和 Raft 的 CPU 核心数量进行扩展。我们的基准测试结果通常表明,ZooKeeper 目前在这方面比 Clickhouse Keeper 做得更好,尽管我们一直在不断提高我们的性能(三个 最近的 例子)。
Keeper 的下一步:Keeper 的多组 Raft,以及更多
展望未来,我们看到了扩展 Keeper 以更好地支持我们上面描述的场景的需求。因此,我们正在通过这个项目迈出一大步——为 Keeper 引入多组 Raft 协议。
因为,如上所述,扩展非分区(非分片)的线性一致性是不可能的,我们将专注于 多组 Raft,我们在其中 分区 存储在 Keeper 中的数据。这允许更多的事务彼此独立(在单独的分区上工作)。通过在同一服务器内为每个分区使用单独的 Raft 实例,Keeper 会自动并行执行独立的事务: 通过多 Raft,Keeper 将能够支持具有更高并行读/写要求的工作负载,例如,拥有数百个节点的超大型 ClickHouse 集群。
加入 Keeper 社区!
听起来很令人兴奋?那么,我们邀请您加入 Keeper 社区。
- 这里 介绍了如何将 Keeper 与 ClickHouse 一起使用
- 要成为 ClickHouse 之外的 Keeper 用户 - 请查看 此 页面,了解何时使用或不使用它
- 这里 是您发布问题的地方;您可以在 X 上关注我们,并 加入 我们的聚会和活动。
我们欢迎对 Keeper 代码库的贡献。请在此处查看我们的路线图 here,并在此处查看我们的贡献者指南 here。
总结
在这篇博文中,我们描述了 ClickHouse Keeper 的特性和优势 - 它是 ZooKeeper 的资源高效的开源即插即用替代品。我们探讨了我们在 ClickHouse Cloud 中的自身使用情况,并在此基础上,展示了一个基准测试套件和结果,突出了 ClickHouse Keeper 在性能指标相当的情况下,始终如一地使用明显更少的硬件资源。我们还分享了我们的路线图以及您可以参与的方式。我们邀请您合作!