关于 TiKV、TiDB、TiFlash 的一些思考

一些常见问题的思考,只代表个人见解。

TiKV 相关

TiKV 写入性能

KV 热点

如果出现热点 Key,机器会吃不消么?写热点是难以避免的。TiKV 选择按 Range 切割,但是 User Key 不跨 Region。一段区间内的写热点,会导致容量超过上限而分裂,新分裂出来的 Region 可以被调度到其他 Node 上,从而实现负载均衡。在文章中提到,可以通过预分区的方式来划分 Region。可是对于单调递增的主键,或者索引,它会永远写在最后一个 Region 上。但我认为热点 Region 未必意味着热点机器,可以先进行 Split,然后通过 Leader Transfer 给其他的 Peer,或者通过 Conf Change 直接干掉自己。我猜测这个主要取决于数据迁移的效率和中心化服务的质量,如果在 Raft Log 阶段就能检测到流量问题并分裂,那么负载有可能被分流到多个相邻的 Region 中。

TiKV 提供了 SHARD_ROW_ID_BITS 来进行打散,这类似于 Spanner 架构中提到的利用哈希解决 Append 写的思路。TiBD 提供了 AUTO_RANDOM 替代 AUTO_INCREMENT。

注意,如果负载是频繁对某个特定的 key 更新,则 TS 一定也被用来计算哈希,不然热点 key 一定是在同一个 Region 内。这样一个 key 的不同版本就分布在不同的 Region 中,就不利于扫表了。因为下推到 TiKV 的请求可以理解为从 [l, r] 去扫出来所有 commit_ts <= scan_ts 的数据,这样的扫表一定是会涉及到所有的机器,性能会很差。对于点查也一样,我们始终要找一个大于 user_key + ts 的 TiKV Key,哈希分片不好 seek。特别地,如果是 SI,那还得扫 [0, scan_ts] 中有没有 Lock,这个过程也要访问多个机器。

如果在构造 key 的时候就进行分片,比如在最左边加一个 shard_id,这样 rehash 会很困难。shard_id 可以比如是通过某个特定字段哈希得到。

在 Spanner 中存在 Tablet,也就是将多个同时访问比较频繁的 Region co-locate,这些 Region 彼此之间未必是有序的,甚至可能属于不同的表。

关于 Region 大小的讨论

较小的 Region 的好处:

  1. 每个 Region 中较低的并发
  2. 更加快速的调度

较大的 Region 的好处:

  1. Placement Driver 的压力变小
  2. CompactLog、Heartbeat 等网络开销变小
  3. 1PC 的事务更多

Raft 存储

原来 TiKV 使用 RocksDB 存储 Raft Log 和相关 Meta,存在几个问题:

  1. WAL + 实际数据,需要写两次盘。
  2. 数据变多,Compaction 负担变大,写放大更大。层数更多,写放大更大。

因此引入了类似 bitcask 架构的 RaftEngine 来解决这个问题。RaftEngine 中每个 Region 对应一个 Memtable,数据先通过 Group Write 写入到文件中,然后再注册到 Memtable 中。在读取时从 Memtable 获取位置,再从文件中读取。因此随着 Region 日志 Apply 进度的不同,RaftEngine 在文件中会存在空洞,因此需要 rewrite。这使得存在一部分 CPU 和 IO 花费在 rewrite 逻辑上,而不能像 PolarDB 一样按照水位线直接删除。RaftEngine 这么做可以减少 fsync 的调用频率,并且充分利用文件系统 buffer 来做聚合。

此外,Raftstore 还使用 async_io 来异步落盘 Raft 日志和 Raft 状态。这样,Raftstore 线程不被 io 阻塞,能够处理更多的 Raft 相关请求和日志。需要注意,这反过来可能会加重 PeerFsm、ApplyFsm 和网络的负担,对 CPU 的要求更高。

日志和数据分离存储

Titan

Titan 的思路是将 RocksDB 中的 value 拿出来存,减少 Compaction 对 CPU 和 IO 的开销,但会带来空间放大。并且数据局部性差,所以范围查询性能较差。

Titan 将这些大 value 有序地存放在一些 blob file 中,并且保存了 value 对应的 user key 用来反查 RocksDB。反查的原因是 blob file 本身需要 gc,所以要通过 user key 来查询是否过期,这会带来一些写放大。

Titan 有两种 gc 策略:

  1. 定时 rewrite blob file
    监听每次 Compaction 事件,从而维护每个 blob 文件中无效数据的大小。每次重写 invalid 率最高的几个文件,并更新回 RocksDB。旧的文件需要确保不再有 Snapshot 引用才可被删除。
  2. 在 LSM-tree compaction 的时候同时进行 blob 文件的重写
    也就是在 Compaction 的同时写到一份新的 blob 文件中。因为不需要的 kv 会在 Compaction 的时候被过滤掉,也就相当于自动完成了 gc。
    这种方案要求 blob 文件也需要伴随着 SST 进行分层,从而带来写放大。并且也有不小的空间放大。因此,该策略只对最下面两层生效。

在有限的场景中,Titan 能够带来收益。

业界也有类似 Titan 的 KV 分离存储方案,比如 WiscKey 等。

TiKV 读取

Cache

TiKV 处理读请求对 Block Cache 要求较高,较低的 Block Cache Hit 会导致读性能倍数下滑。Block Cache 需要占用接近一般的内存,但也需要保留一部分给系统作为 Page Cache,以及处理查询时的内存。

不同的压缩方式,对 CPU 的压力不同。

Multi Raft 相关

关于 Raft 协议本身

Follower Replication 和 Follower Snapshot

Follower Snapshot 的好处有:

  1. 因为是有处于一个 Zone 的 Follower 发送 Snapshot,所以可能更快。并且跨 Zone 流量也少
  2. 减少 Leader 的负担

TiFlash 做了 Learner Snapshot,相比 Follower Snapshot,它甚至是一个异构的 Snapshot。CRDB 做了类似的工作,称为 Delegate Snapshot。TiKV 目前还不支持。

关于读

Raft 的一个问题就是读的时候无论是 Leader 还是 Follower 都需要 Read Index。比如,对 Leader 而言,它需要问 quorum 自己当前是否还是 Leader。TiKV 一般 Leader Read 提供两种方案,第一种是 read_local,也就是 Leader 节点上 lease 读,另一种是 read_index,也就是在不确定自己是否还是 Leader 的时候,进行 ReadIndex。

Raft 状态的思考

RaftLocalState 中相比 Raft 协议多包含了 last_index 和 commit。其中 commit 可以避免重启后不能立即 apply 的情况。

存储 Raft 状态和 Region 状态

TiKV 使用 Raft Engine 存储 Raft 元信息和 Raft 日志。使用 KV Engine 存 Region 信息、Region Apply 信息和具体的 KV数据。

一个 Eager 落盘导致的问题

并不是所有时候,eager 落盘都能保证正确性问题。下面就是一个例子。
前面说过,在 TiKV 的实现中有两个 engine,KVEngine 存储 KV Meta 和 KV Data,RaftEngine 存储 Raft Meta 和 Raft Data。其中有一个 Apply Snapshot 的场景会同时原子地修改这两个 Engine,但可惜这两个 Engine 无法做到原子地落盘。并且因为两个 Engine 中都存有 Meta 和 Data,所以任意的先后顺序,都会导致数据不一致。这里的解决方式是将 RaftEngine 中的的 Raft Meta 写到 KVEngine 中,称为 Snapshot Meta。写入的时候,会先写 KVEngine,再写 RaftEngine。当在两个非原子写入中间出现宕机,从而不一致的时候,会使用 KVEngine 中的 Raft Meta 替换 RaftEngine 中的 Raft Meta。

Apply Snapshot阶段开始时,它会调用 clear_meta 删除掉 KV Meta、Raft Meta 和 Raft Data,但这个删除是不应该立即落盘的,而是在 WriteBatch 里面。在这之后,还会再往 WriteBatch 中写入 Snapshot Meta 等。这些写入会被一起发送给一个 Async Write 写入。我们的错误是,在实现删除 Raft Engine 数据时,并不是写 Write Batch,而是直接写盘。在 clear_meta 之后系统又立即宕机了。这样重启恢复后,就会看到空的 Raft Meta 和 Raft Data,但 KV Meta 却还存在。这是一个 Panic 错误,因为两个 Meta 不一致了。

这样的错误是难以调查的,我们可以加日志获得重启后从磁盘中读到的结果,但仍然不知道这个结果是如何被写入的。查的方式是脑补,也就是针对这样的场景,假设在不同时刻宕机,考虑会出现什么样的持久化状态。
这里,KV Meta 的落盘信息是有的,它可能是没清就宕机了,也可能是写完新的数据之后宕机的。考量这个可以看一些 Meta 信息有没有写入,比如我们发现 Snapshot Meta 并不存在,因此说明是前一种情况。既然如此,为什么 Raft Meta 和 Data 都没了呢?只能说明是 Raft 的清早了。

当然,这里有个迷惑点,就是 KV Meta 提示当前是在 Applying Snapshot 状态,而如果我们是第一种情况的话,这个 Applying 状态应该还没有被写入。这个原因是这个实例发生了多次重启,在 T-2 次启动后 Apply Snapshot 时,KVEngine 和 RaftEngine 都落盘成功了,但是后续的流程没进行下去就重启了。所以在 T-1 次启动会重新 Apply Snapshot,但这一次甚至没到落盘就重启了,而 Snapshot Meta 是金标准。然后就是我们见到的 T 次启动的错误。这启示我们不能只通过一个元数据来判断当前集群的状态,而是要检查所有的元数据,来石锤当前状态是如何得到的。

Multi Raft 的思考

共识层和事务层的关系

Percolator 事务提交模型中,commit_ts(R) < start_ts(T) 的事务 R 对事务 T 可⻅。不满⾜该关系的事务为并发事务,并发事务如果访问相同的 key 将会导致其中⼀个事务会碰到 Lock ⽽回滚。
Raft 的 Read Index 模型中,一个读请求需要等到 applied_index 大于等于 read_index 时,才能读取数据。但并不保证是否能读到 applied_index = x + 1 时的数据。实际上无论是否读到,都不违背强一致读的原则。因为如果一个读 A 能读到 applied_index = x + 1,而另一个读 B happen after 读 A,那么读 B 一定会读到 applied_index >= x + 1 的数据。

TiKV 的共识层在事务层之下。在事务 Commit 之前的很多数据也会被复制到多数节点上,这产生了一些写放大。但也需要注意其带来的好处:

  1. 共识层实际为 Percolator 提供了类似 BigTable 的存储。
    首先提供了外部一致性。
    然后提供了 PUT default/PUT lock 和 PUT write/DEL lock 的原子性写入。
    当然,这里要先读后写,可能会有 Write Skew。
  2. 共识层本身也可以作为一个 Raw KV 对外服务。
  3. 共识层参与定序。这个在后面介绍。
  4. 多个 Raft Group 组成的共识层提高了并发能力。
  5. Lock 的存在性和⼀致性由该⾏所处的 Raft Group 保障。
  6. 事务提交后,会写⼊ Write 并删除 Lock,其原⼦性由 Raft Write Batch 保障。
  7. 共识层提供了全序广播语义。
    “在 xx 之前,一定不会有别的 Lock 和 Write 了”

当然这也存在一个 argue 点,因为 Raft Log 本身也是 total order 的。虽然我们目前不是全局一个 Raft Group 的,但看起来会有一些冗余。后面会讨论。
特别地,在 CDC 服务和 TiFlash 中,我们实际上不会处理未 Commit 的数据。

共识序和事务序

双重定序

事务层的实现中,为了满足隔离性,通常会给事务分配 id 来表示相互依赖的事务之间的偏序关系。TiDB 中使用了 TSO,Spanner 中使用了 TrueTime,CRDB 中使用了 HLC。
共识层的实现中,为了实现容灾和高可用,使用共识算法在各个 RSM 之间复制日志,这些日志为全序关系,RSM 可以应用这个全序关系确保所有副本间是线性一致的。

事务层生成 TSO 和共识层生成 Log 两个行为:

  1. 不是原子的
  2. 也不构成全序关系
    实际上也没必要,两个不相交的事务按照事务序本来可以并行 Commit 的,但因为要写到共识层,必须又要排出一个全序关系来。
  3. 甚至一个事务的 commit_ts (相比某个特定事务)更小,而 index 更大
    下面会展示这种情况,并详细阐述。

总而言之,Percolator 协议保证了事务层能够生成一个特定的排序,并且按照它的二阶段方式写入到共识层。共识层保证了所有的副本都会应用该特定排序。

共识层为事务层提供帮助

目前 TiKV 通过一个 pd 分配一个全局的 tso 来作为事务的 start_ts 和 commit_ts,所以它们之间彼此构成全序关系。通过 start_ts 和 commit_ts 可以构建有依赖的事务之间偏序关系,也可以用来判断事务是否是 concurrent 的。如果在单个节点上串行地 commit 这些事务,则面临问题:

  1. 整个系统毫无并行度
    这个应该算是 MultiRaft 的一个 bonus,正如后面讲的,如果没有 MultiRaft,同样可以做 partitioning。
    因此,TiKV 在多个线性一致的存储(Region)上储存这些事务,它保证了每个事务在每个 Region 上都遵循了 start_ts 和 commit_ts 所 imply 的顺序,也 Percolator 那一套。这样尽管各个 Region 之间是并发的了,但只要 Region 内遵循这个 order 就行了。

    当然,这个切分也未必是按照 Region 来,比如 CDC 会使用表来切分。无论按照哪种方式来切分,我觉得一个实现的要点是每个 shard 在调度上是不可以再分的了。比如一个 Region 的一部分数据在 store 1 上,另一部分数据在 store 2 上,这样做实际上会导致无论在 store 1 和 store 2 上都很难独立构建出该 Region 上数据的全序关系,比如 store 1 如果不和 store 2 交互,那么就很难知道 store 2 上还有没有 happen before 它的事务了。比如说,如果两个 store 上 apply 这个 Region 的 log 的进度不一样。

  2. 如何判断某个 tso 之前还有没有其他 Lock 或者 Write?
    因此,读事务会在取得 start_ts 后,再通过 ReadIndex 请求一下 Region Leader 上的 commit_index。那么假设在这之前 Region 上有写入任意的 Lock 或者 Write,都能被 ReadIndex 扫到。这样就保证了读事务能看到 start_ts 之前的所有修改。至于 start_ts 之后的也有 Lock 可以帮忙。
    同样考虑一个Snapshot Isolation(SI)/一个两难问题,这里不再详细展开具体内容。但 ReadIndex 提供了一个保证,就是截止到 read_index,这个 Region 上到底有没有 Write,是很确定的。我理解这实际上就是一种全序广播了。破坏这种全序广播可能会有严重后果,比如如果将 Write 乱序到 Lock 前面,则违反了 Percolator 事务的约束。我们实际上也没办法很好的处理,在“跨 Region 提交事务”中,就构造出了这样的场景。

此外,对于并发事务,共识层也会对它们之间排出一个串行的顺序,比如两个并发的事务不能同时 Commit,而要等到 Log 按序 apply 而这可能有点过强。诸如 ParallelRaft 或者 MultiPaxos 的算法允许并行 apply,可以解决此问题,但会导致 Leader 和 Follower 之间的 apply order 难以统一,从而无法实现 Follower Read。

共识层对并发事务的乱序

刚才说过,共识层未必会按照事务序写入。这也很容易理解,因为取 start_ts 和 commit_ts 和真正写共识层不是原子的。
TiKV 事务在读取时,需要同时接收事务层和共识层的定序。为了满⾜线性⼀致读,需要⾸先带上 start_ts,发送⼀个 ReadIndexRequest 给对应的 Region,求出⼀个 applied_index。在实际实现中,start_ts 并⽆作⽤。
如下所示,Key a 和 Key b 属于两个事务。在事务提交前,可以看到或者得到的保证是:

  1. start_ts(a) < commit_ts(a)
  2. start_ts(b) < commit_ts(b)
  3. start_ts(a) < start_ts(b)
  4. 并且这两个是并发事务,也就是说 commit_ts(a) > start_ts(b)

不妨假设 commit_ts 分别为 4 和 6,然后再假如以 (start_ts=7, read_index=202) 读取,则可能读到一个锁和 Key b,或者读到 Key a 和 Key b,前者需要 ResolveLock,实际上导致以新的 read_index 来读取。因为 Key a 和 Key b 的写入对 start_ts=7 的读取事务可见,但该读取可能在 applied_index 大于等于 read_index 的任意时刻返回读取的值。因为共识层的序至少保证了同一个 key 的 prewrite 在 commit 前面。

  1. Key a:(start_ts: 1, applied_index: 100), (commit_ts: 4, applied_index: 210)
  2. Key b:(start_ts: 3, applied_index: 101), (commit_ts: 6, applied_index: 200)

反过来讲,如果共识层给出下面的顺序,我们看到了中间的 a 或者 b 上有锁。因为这两个事务是并发事务,所以这也是 OK 的

  1. Key a:(start_ts: 1, applied_index: 100), (commit_ts: 4, applied_index: 200)
  2. Key b:(start_ts: 3, applied_index: 101), (commit_ts: 6, applied_index: 210)

可以看到,尽管将事务拆到了 N 个线性一致的存储上执行,并且这些存储可能对并发事务任意定序,但最终读到的结果还是满足了线性一致,以及事务隔离层的要求的。

跨 Region 提交事务

TiFlash 不能在看到第一个 write 记录时“提交”该事务的所有 key,这里的“提交”指的是写入下层存储,比如将 Default 写过去,但并不包含删除 Lock 等。
现在比如考虑两个事务,假设 a 在一个 region r1,b 和 c 在另一个 region r2 让:

1
2
commit a(applied_index@r1=100), commit b(applied_index@r2=300), commit_ts=4
commit c(applied_index@r2=200), commit_ts=1

从事务层上来看,一定有读事务能看到 c,或者 a、b、c。现在如果看到 a 提交了,能不能跑到 b 的 region 上把 b 也提交了呢?我认为是不可以的,因为从共识层上来说,b 在 c 的后面被 commit 的,如果用 (start_ts > 4, read_index = 250) 去读的话,可能读到 lock b,甚至可能连 lock b 也还没被写入,当然也有可能读到 b。但如果我们在 apply a 的 write 记录的时候发现了 a 被 write 了,就直接写 b 的 write 记录,那么就导致 b 一定在 c 前面就能被读到,实际上违反了共识层的序。

具体来说,不妨考虑 client 先后从 Learner 和 Leader 读:

  1. 在 Learner 上,它使用 read_index = 250 读,但是因为 commit a 已经被 apply 的原因,所以它一定读到了 commit b。
    当然细究下来,因为 lock + default 是原子的,所以实际上 write 无法被正确执行。但这就是 orphan write key 的问题,之前在处理 multi rocks 的时候就解过,我觉得很复杂。在这个场景下,我觉得免不了要去进行等待。在异步系统中的等待,我觉得可以理解为是一种活性问题。
  2. 在 Leader 上,此时 Leader apply 到了 260,所以此时 Leader 上一定没有 commit b,这导致它读不到 commit b。

这里线性一致读就被破坏了。反之,如果按共识序 commit,则不会有这种情况。具体就不展开了。

这个场景在单 Region 上无法构造,原因是单 Region 上是串行的。尽管“在看到第一个 write 记录时‘提交’该事务的所有 key”可能相当于让一部分 Write 被乱序,但这种乱序不是直接去把 Write 挪到 Lock 之前那样是破坏性的。比如说,因为 Percolator 的特性,单 Region 上的某个事务的 Prewrite 一定都在 Commit 前面。因此,就算在看到第一个 Write 时候,将该事务的所有 Default 都提前写到下层存储,也不至于提前到某个 Lock 前面。这样被写的 Key 始终有 Lock 保护,直到看到它对应的 Write。
而在多 Region 中不同 Region 可以说是完全异步的(不考虑 Split 等),那我就可以构造一个无比提前的 Write,让它失去 Lock 的保护。

Split/Merge 和事务

Split/Merge 和 Read

Split 和 Merge 会导致 Region 发生变化,自然也可能会影响读取。主要体现在下面几个方面:

  1. 影响 Lease 本身或者 Lease 续约
  2. 推高 RegionEpoch 从而导致 ReadIndex 失败

Split/Merge 和 Apply Snapshot

Multi Raft 实现的复杂度,很大程度在处理 Split/Merge 和 Apply Snapshot 的冲突上。

Split 和 Apply Snapshot 的冲突

我们需要处理一个 Region 上的 Follower 还没有执行到分裂为 Base 和 Derived 前,一份来自 Derived 的 Snapshot 已经被发过来的情况。这会产生 Region Overlap 的问题,在一些下层存储中会导致数据损坏。一种方案是在 Base 完成分裂前根据 Epoch 拒绝掉这些 Snapshot。

Merge 和 Apply Snapshot 的冲突

Merge 过程可以简单理解为下面几步:

  1. 调度 Source 和 Target Region 的各个 Peer,让它们对齐到同一个 Store 上。
  2. Source Peer 执行 Prepare Merge。
  3. Source Peer 等待 Target Peer 追完 Source Peer 的日志。
  4. Source Peer 对 Target Peer 去 Propose Commit Merge。
  5. Target Peer 执行 Commit Merge。

可能在下面一些阶段收到 Snapshot:

  1. Prepare Merge 结束
  2. Leader 上的 Commit Merge 结束,但 Follower 上的 Commit Merge 还没有开始

Split 和 Generate Snapshot 的冲突

主要指 Split 等会改变 RegionEpoch 从而导致 Snapshot 失效。

Raft Group 和 Data Range 的对应关系

TiKV 中,Raft Group 和 Region 严格一一对应。TiKV 中 Region 管理一段范围内的数据,在其他一些实现中,这段范围可能被称作 Shard、Partition 等。讨论下这个设计:

  1. Raft 本身和 Region 数据的版本引入了全序关系
    首先,Raft Admin Command 会穿插在写入之间形成很多 barrier,带来额外的持久化负担。
    然后,这导致了新创建的 peer 只能通过 Snapshot 追进度的情况。从 Raft 协议来看,ConfChange 之前的日志的提交和复制应当遵守 C_old 的配置项目,但是它并没有禁止进入 C_new 状态的 Leader 给新 peer 发送 ConfChange 之前的日志。但考虑到如果新 peer 还在处理 C_old 时代的日志,它的本地状态比如 RegionLocalState 肯定对应了 C_old,这个时候它接受到了一个“不认识”的 store 的 AppendEntries,这是比较奇怪的。
  2. Raft Group 不稳定
    Split 会分出独立的 Raft Group,给 pd 调度带来压力。也变相增大了 recover 的工作量。
    Merge 两个 Region 会销毁一个 Raft Group,这里面有不少 corner case。比如 Leader 关掉后的孤儿 Learner 问题。

我觉得可能 Spanner 的架构会更好一点。也就是说:

  1. 一个 “Spanner Region” 一个 Raft Group,但这个 “Spanner Region” 不再和某个 Key range 绑定。
  2. 一个 “Spanner Region” 下可以被调度多个 Key range。例如有局部性的 Key range 可以被调度在一起,或者处于打散负载的目的可以将 Key range 进行随机的分布。

Raft Group 和 Data Range 分开的架构

在这之上还有一些设计:

  1. 全局需要维护多少个 Raft Group?
    一个 Raft Group 可能需要处理不同 Key range 的数据。但全局关系肯定是过强了,破坏了 Partitioning 的初衷。所以会更倾向于引入乱序 Apply 机制来提高 RSM 的吞吐量。
  2. 谁有权限写 Data Range?
    一般来说,会将对应的 Raft Group Leader 设置为 Data Range 的 “Leader”,让它来处理写入。这样做的好处是可以减少一次 RPC。

另外,分开的实现还有个好处,就是如果 Raft 层的 Leader 发生切换,Data Range 层的读取不会收到影响,而是可以 bypass 掉 Raft 层。CRDB 就是这样实现的,也就是类似是 Data Range 上的 LeaseRead。相比之下,TiKV 的 LeaseRead 和 Raft Leader 的生命周期是绑定的。

另外值得一提的是 CRDB 将 Lease 和机器绑定而不是和 Data Range 绑定,从而减少网络开销。它的做法是每个 Data Range 的 “Leader” 会去维护一个 meta 表(也是一个 Data Range)上的 liveness 记录。我理解它可以以一个比较低的频率去更新 liveness 记录,因为如果不是节点挂了下限,或者是重新调度到当前 Raft Leader 的节点上这两种情况,只要 Raft Leader 还是同一个,那么就完全没有必要续期。而 TiKV 的绑定方式则必须要求 Lease 是比 Election Timeout 要短的。对于 meta 表自己的 Lease,是通过 expiration time 来维护的。

在本文的后面还会提到 Follower Read 相关的话题,特别是它和乱序 apply 的关系。我个人觉得,如果将 Data Range 和 Raft Group 分开,我们仍然是可以实现 Follower Read 的。如果你把 Data Range 看成一个 RSM,那这种架构就类似于一个 Raft Group 去管理多个 RSM。我们在 Data Range 上维护一个 index,应该就行了。

Raft 日志的内容

Raft 日志中到底记录什么呢?可以看下面的总结:

  1. TiKV
    TiKV 中 Raft 日志分为 Admin 和 Write。Admin 基本只和 Raft 和 Region 管理有关。Raft 指的是 Raft 的成员变更,比如 Add/Remove Voter/Learner,TransferLeader 等。Region 指的是管理的 key range 的元数据变更,比如 Split、Merge、数据校验等。
    Admin 和 Write 在一起构成全序关系,这个话题之前已经展开讨论过了。
    Write 包含 Put、Delete、DeleteRange 和 IngestSST,这些都是逻辑日志,或者说是不 aware 下层 rocksdb 的。
  2. OceanBase
    OceanBase 中复制的是 clog。从文档来看,它们复制的是物理日志。通过 replay clog,能够得到同样的 log 文件,其中记录的是 redo log。
    下面来自Oceanbase 文档

    OceanBase 数据库单台物理机上启动一个 observer 进程,有几万到十万分区,所有分区同时共用一个 Clog 文件,当写入的 Clog 文件超过配置的阈值(默认为 64 MB)时,会打开新的 Clog 文件进行写入。
    OBServer 收到的某个分区 Leader 的写请求产生的 Clog、其他节点 OBServer 同步过来的 Clog(存在分区同在一个 Paxos Group),都写入 Log Buffer 中,由单个 IO 线程批量刷入 Clog 文件。

  3. PolarDB
    在《PolarFS: An Ultra-low Latency and Failure Resilient Distributed File System for Shared Storage Cloud Database》中讲得比较清楚。
    PolarDB 的存储层基于 PolarFS,计算节点共享地访问这个存储层。PolarDB 中每个数据库对应 PolarFS 中的一个卷,每个卷由若干 Chunk 组成。不同于 TiKV 的 Region,这里 Chunk 大小为 10GB,而卷的大小在 10GB 到 100TB 之间,所以它们元数据节点的调度压力会小很多,并且所有节点的元数据都可以缓存在内存中。一个 Chunck Server 管理多个 Chunk,PolarDB 通过增加 ChunkServer 的数量来平衡热点。这里我觉得 TiKV 的 multi rocks 方案可能更好,因为它允许一个 hot region 被分裂。在 PolarDB 中,一个服务器上运行多个 ChunkServer,但每个 ChunkServer 对应一个专用的 SSD,并且绑定一个专用的 CPU 核心。
    一个 Chunk 由 64KB 大小的 block 组成。PolarFS 的 Raft 日志实际复制的是这些 block 的 WAL。
  4. Kudu
    Kudu 中复制的是逻辑日志。他们的观点是这样可以实现各个 Replica 在存储格式上是解耦的。

进一步讨论:日志和选举的关系

Raft 中的领导人完全性原则要求 Leader 必须拥有所有已提交的日志,这实际上是一个比较强的约束。在 Ongaro 等人对于 MultiPaxos 的描述中,可以发现该约束是可以被消减掉的,从而选举过程可以不关注日志的完备性。
在此基础上,可以让选举体现出其他的优先级。以 Ob 的 Palf 为例,它的“一呼百应”的方案,可以始终给距离自己最“近”的节点投票。而 Raft 选举的实质是谁状态更新,谁就更容易当选。这个方案目前来看,无论是否效果最优,但确实代价比较大。

有关 Raft 的日志和选举关系的讨论,可以见 Raft 算法介绍中的“日志和选举”章节 详细讨论。

进一步讨论:日志和事务的关系

将多个分区的写入统一到一个 Raft Group 中进行复制,应该是有利于事务的。因为如果一个事务跨 Region,就会是一个分布式事务,而如果只有一个 Raft Group,那么就不会涉及到跨 Region 的问题。

Mono LSM 和 Multi LSM 的考量

这里指的是不同的 Region 的数据是否 share 一个 LSM 树。我认为如果使用 range partition,那么 multi lsm 的策略是一个非常重要的优化。

线性一致读

Follower Read

TiDB 支持多种读取方式,例如最近 Peer、Leader、Follower、Learner、自适应等多种模式,这些依赖于 Follower Read,在这之前都需要从 Raft Leader 读取。

不同于 ParallelRaft 和 MultiPaxos 的部分实现,TiKV 会串行地 apply raft log。

  1. 这样的好处是,更容易通过 Read Index 实现 Follower Read 了。TiKV 在这一点上行得通,主要还是因为它的数据和 Raft Group 绑定的缘故。也就是以 scheduler 为代价来实现 Partitioning,从而减少各个 Raft Group 的压力。
  2. 这样的坏处是,引入了更强的全序关系。因为我们实现共识层的目的是服务上层的事务层,而事务层本身就允许并行事务以任意的顺序被提交,所以在共识层排成强序,实际上是多余的。当然,Partitioning 分成多个 Raft Group 能减少这部分的强序关系的数量。

总的来说,TiKV 实现的 Follower Read,是通常被称作 Strong Follower Read 的类型。

Learner Read

不同于 Follower,Learner 不是 Voter,没有选举功能。所以 Learner Read 和 Follower Read 有不同。
Learner Read 在 TiFlash 场景下更为丰富,在 TiFlash 章节讨论。

强一致读(加上事务)

从共识层上来讲,强一致,或者线性一致有明确的定义。CRDB将其“推广”到事务层之上,也就是归结到所谓的 non-stale 读上。因为 CRDB 只有 leaseholder 也就是所谓的 Leader 能服务读。但推广到有 Follower Read 的场景下就是,在任意的节点上:

  1. 在 SERIALIZABLE 下,读事务应该能看到在它之前已经提交了的所有的写事务。这里的“它之前”我理解根据事务的实现的不同而不同,但至少要在事务的第一个读之前。比如 Percolator 模型中就是 start_ts。
  2. 在 RC 级别上,事务中的每一个读语句能看到在它之前已经提交了的所有的写事务。

Stale Read

Stale Read 的作用是让读请求被分配到任一节点上,从而避免某热点机器,或者跨数据中心的 read index 请求产生的延迟。

这样的事务只能服务读,并且 staleness 也是需要被严格控制的。

Stale Read 是读 ts 时间点上所有已提交事务的旧数据。因为读不到最新的写入,所以不是强一致的。但它仍然保持有全局事务记录一致性,并且不破坏隔离级别。我理解可能就是所谓的 Time travel query。

一般提供两种:

  1. 精确时间戳
  2. 有界时间
    在给定的时间范围内选择一个合适的时间戳,该时间戳能保证所访问的副本上不存在开始于这个时间戳之前且还没有提交的相关事务,即能保证所访问的可用副本上执行读取操作而且不会被阻塞。
    因此这样的读取方式能提高可用性。

使用 Stale Read 需要 NTP 的支持。

所以它并不是“弱一致读”,无论从哪一个节点返回的结果都是一致的,不会出现 A 返回 1000 笔记录,而 B 返回 1111 笔记录的情况。

事务相关

加锁的时机

无论是悲观锁还是乐观锁,都面临加锁时机的选取。

在提交时加锁存在下面的问题:

  1. 乐观锁的问题
  2. 因为整个事务需要缓存在内存中,所以大事务面临 OOM

在 DML 时加锁存在下面的问题:

  1. 每写一个 key 都要和 TiKV 通讯一次
  2. 多次对同一个 key 的 prewrite 无法确认先后(网络可能被任意延迟)
  3. 对 TiFlash 而言,因为列需要按照 commit_ts 排序,所以最好等到 commit 之后再行转列,而 DML 加锁意味着 DML 阶段 prewrite,那么在 DML 阶段就可以行转列了

Percolator 事务和共识层乱序

在什么程度上共识层可以乱序呢?我的结论是:

  1. 跨 Region 情况下会破坏线性一致读,并且从事务层修正的难度比较大,可能引入很长的等待
  2. 单 Region 上,如果保证 Lock 和 Write 的全局序,但只在发现事务 A 的第一个 commit 的时候,将事务相关的所有的 Default 写入,这种情况应该是可以的。对于较为基础的 case 我有 tla 证明
    根据具体实现,需要落盘 Default 和 Lock 是一起的,比如先落盘 Lock 再落盘 Default。可以不用原子落盘两个 cf。

Partitioned RaftKV 相关

和 Mono-store RaftKV 的兼容性问题

新架构简化了 Snapshot 的生成和注入流程:

  1. 在生成时,只需要对当前 Region 对应的 RocksDB 做一个 Snapshot 就行。这个 Snapshot 包含的数据可以新于 Raft Local State。
  2. 在注入时,只需要重命名 RocksDB 文件夹即可。不需要处理 range overlap 的问题。因此不需要引入单线程的 region worker。

因此 Mono-store RaftKV 需要处理下列问题:

  1. RocksDB 数据和 Raft 状态不一致。
  2. Snapshot 的 Range 可能和其他本地 Region Overlap。

不光是 Snapshot,在 Partitioned RaftKV 中,Region Peer 之间也可能互相 Overlap。所幸这个场景只会出现在 BatchSplit 和调度 Peer 发生冲突的情况下。

在新架构中,Apply 的落盘也实现了异步化,现在下层引擎可以选择在任意时刻落盘数据,并且在落盘完毕后通知 raftstore。这对 TiFlash 来说是一件好事,我们可以由此来让 KVStore 的落盘不再阻塞。

采用更大的 Region 的性能影响

  1. 可以采用 Parallel Raft 的方式实现并行 Apply。
  2. 单个 Region 的 Apply 压力会增大,但是下层 RocksDB 的负担减轻了。相比于单个实例的 RocksDB,新架构的层数更少,并且并发写入也更少。后续还可以尝试支持多盘部署。

另一个考量点是如果集群中出现很多小表,那么大 Region 的效果不能完全展示:

  1. 因为编码的问题,table 编码不相邻的表不能被合并到同一个 Region 中。
  2. 相邻的 table 合并会给 TiFlash 带来不少问题。例如如果给一些小表添加 TiFlash 副本,并且这个小表被合并到一个大 Region 中,那么发来的 Snapshot 可能非常大,并且包含了大量 TiFlash 不需要的数据。此外,TiFlash 本身的存储引擎也需要做出调整。

TiFlash 相关

架构

为什么 TiFlash 实现 HTAP 基于 Raft?

Raft 帮助我们实现:

  1. LB
  2. HA
  3. Sharding

但是 TiFlash 只通过 Raft 同步各个表的 record 部分的数据。我们不同步索引,因为不需要。我们不同步 DDL 相关结构,因为并不是所有表都存在 TiFlash 副本。取而代之的是在解析失败,或者后台任务中,定期取请求 TiKV 的 Schema。

另一种强一致的方案是基于 CDC 和 safe TS,这样的方案理论上达不到和 Raft 一样的性能。这是因为类似 CDC 的方案的 safe TS 是基于表的,而 Raft 的 applied_index 是基于 Region 的。在一些场景下,如果一个 write 涉及到多个 Region,那么为了保证原子性,需要这些 Region 上的数据全部被写完,才能前进 ts,这会影响大事务的同步效率。另外,在读取时,也需要等待 safe TS 前进之后,才能读取。而基于 Raft 的方案只需要相关的 Region 的 applied_index 前进到 ReadIndex 就可以了。另外,CDC 也只保证单表事务。

为什么在 TiSpark 之外还开发 TiFlash

TiSpark 直接操作 TiKV,绕过了事务层,可能产生一致性问题。
TiSpark 没有自己的列式存储,处理分析性查询并不占优势。

TiFlash 是副本越多越好么?

不是。理论上是 1 副本的性能最好,但是考虑到高可用,通常建议 2 副本。

1 副本性能最好的原因是,DeltaTree 的 Segment 的粒度要显著比 TiKV 的 region 大,因此同一个 Segment 上会存在多个 Region。

考虑存在 4 个 Region,从 A 到 D,如果只设置一个副本,其分布类似

1
2
Store1: [A0, B0]
Store2: [C0, D0]

而如果设置两个副本,其分布类似

1
2
Store1: [A0, B0, C0, D0]
Store2: [A1, B1, C1, D1]

假如一个查询同时覆盖这 4 个 region,那么一副本的情况下,Store1 和 Store2 分别扫描自己的一部分数据就行了。而两副本的情况下,则可能扫描到多余的 Region 的数据。

有一些人还会觉得副本数越多,并发能力越强。但在基于 Raft 的分区策略下,并发能力是通过合理的 Sharding 来提升的。而具体到一个副本上是可以支持大量的并发查询的,并且我们也更容易对这些查询做 Cache,当然在 AP 场景下可能有限。

DDL 如何同步?

TiDB 的 DDL 的优化点:

  1. 延迟 reorg 到读
    例如 add column 的 reorg 阶段实际上不会写入默认值,而是在读的时候才返回默认值。
  2. 以新增代替变更
    例如 alter column 只会扩大列的值域,比如 int8 扩大为 int64。如果涉及缩小至于或者改变类型,则会体现为新增一个 column,然后把老的 detach 掉。
    因此新的 Schema 能够解析老的 Schema。

TiFlash 上 DDL 的特点:

  1. TiFlash 只需要同步需要表的 DDL。
  2. TiFlash 只需要同步部分 DDL 类型,诸如 add index 等 DDL 并不需要处理,更没有 reorg 过程。
  3. 尽管 TiDB 将 schema 存在 TiKV 上,但 TiKV 是 schemaless 的。所以如果 TiFlash 只从 TiKV 同步数据,就会涉及解码等工作。

因此,TiFlash 有两种 DDL 同步方式:

  1. 定期拉取(一般是 10s)并更新
    根据 TiFlash 和 TiDB 上 version 落后的情况,可以分为拉 diff 和拉全量。
    该方式已经能解决大部分 drop table 的问题了。但通过该方式无法保证当前任意时间点上的 schema 一定和 TiDB 是一致的,所以一定存在解析失败的情况。
  2. 当解析 row 失败的时候更新 schema,称为 lazy sync

在更新之后,TiFlash 会自己维护一份 schema。

这里面存在的问题主要是两种 DDL 同步方式和实际 raft log 是异步的。因为 TiDB 和 TiFlash 的特点,这个异步是可以被处理的,并且尽可能去掉全序的依赖是很多系统的设计理念,所以这种做法本身也是挺好的,但其中 corner case 很多。例如:

  1. Schema 和 row data 中的列数对不上。这种情况无论是谁缺,至少可以通过拉一次 Schema 来解决。有些场景甚至可以不拉 schema。
  2. 某个列的类型变了
  3. 一张表 drop 后,TiKV 中就无法读取该表的 schema 了。如果在 drop 前有一条 add column,但 lazy sync 又没有读到,那么 TiFlash 就看不到。所以如果后续有一条 row 写入过来,TiFlash 就会丢弃这个 column。假如这个 table 被 recover 了,那么 TiFlash 就会读不到这个 column 的数据。
  4. 一张表对应的 DeltaMerge 实例创建前,这张表就被 drop 掉了。在此之后,row 数据到来,并导致 DeltaMerge 实例被创建。

TiFlash 的高可用

对于复制自动机的系统,高可用主要取决于选举的速度。
对于 TiFlash 来说,它不参与选举,但选举本身同样会有影响,一方面是 ReadIndex,另一方面是无主的时候无法复制日志。但除此之外,TiFlash 自身的宕机和重启也影响高可用。因为一个批量查询会被下推给 tiflash,以避免影响 TP,如果此时 TiFlash 没追上,则查询会 hang 住。所以 TiFlash 的高可用还和追日志的规模有关。

Raft 共识层

有关 Learner 的问题

Peer 活性

Learner 尽管在 Raft Group 中,但不参与投票。所以当 Voter 节点因为 Region 被销毁(通常因为 merge)全部被销毁后,Learner 节点就无法找到 Leader 节点。对于 Voter 节点来说,这种情况它可以发起选举,然后发现其他节点上的 Tombstone 标记,从而确认 Region 已经被摧毁了。但因为 Learner 不参与投票,所以是无法发现这种情况的,从而僵死。这给 TiFlash 带来了不少 Corner Case:

  1. 在 Region 销毁的场景如 CommitMerge,target region 的 Voter 至少可以在 Leader 销毁之后,因为超时触发选举,从而启动自毁。而 Learner 则不行,会 miss leader 然后卡死
    特别地,CommitMerge 本身对 Source Peer 也会有检查,所以这里可能造成连环等待。比如如果在等待 Source 追数据,就会 Yield 为 WaitMergeSource。如果卡在 CommitMerge 上,那么后续的 RemovePeer 也无法执行。
  2. 在 ConfChange 中,如果删除了某个 Learner,但又没有能够将该日志复制给 Learner,那么稍后 Learner 就不会得到 Leader 的任何消息,从而一样卡死。
  3. 在 BatchSplit 中,如果新 Split 出来的 Region 在 TiFlash apply BatchSplit 命令前就在所有 Voter 节点中被删除的话,后续 TiFlash 节点即使 apply 完 BatchSplit,也无法再收到任何日志,因为 Leader peer 已经不存在了

上述的卡死在之前需要等待 2h 之后触发存活性检查才会被发现。或者人工将僵死的 Region peer 设置为 tombstone 状态。

Snapshot

另外,Raft Log GC 也需要 respect Learner 的进度,不然会导致频繁的 Snapshot 生成失败。

有关 LearnerRead

由 Follower Read 派生出来的 Learner Read 也让 TiFlash 成为一个强一致的 HTAP。

存储

为什么在列存前还有一个 KVStore?

在 CStore 模型中,WS 和 RS 都是列存,WS 的数据通过 Tuple Mover 被批量合并到 RS 中。体现在 TiFlash 中,WS 是 DM 中基于 PS 的 Delta 层,而 RS 是 Stable 层。

除此之外,TiFlash 还有一个 KVStore,目的是:

  1. 保存未提交的数据,并实现 Percolator 事务的部分功能
    因为只有已提交的数据才会写入行存,为了和 Apply 状态机一致,所以未提交的数据同样需要持久化,因此引入 KVStore。
  2. KVStore 管控 Apply 进度,对 DM 屏蔽了上游。DM 可以异步落盘。日志复制的架构下,上游的落盘进度不能比下游更新,因为下游更新,重放是幂等的;而上游更新,会丢数据。

为什么不将未提交的数据直接写在列存中呢?

  1. KVStore 需要负责维护 apply 状态机
    当然我们可以将这一部分作为单独的 Raft 模块,所以这不是很 solid 的理由。
  2. KVStore 不仅是一个容器,还是 Percolator 事务的执行器
    例如,它需要维护当前 Region 上的所有 Lock。在一个查询过来时,需要检查该查询是否和 Lock 冲突,并尝试 resolve lock。而在列存中维护 lock cf 会很奇怪。
  3. 这意味着要执行近乎实时的行转列
    首先,如果存一些未提交数据在 KVStore 中,然后在提交时 batch 执行行转列,有可能可以只读取一次 schema 结构,减少开销。
    其次,TiDB 中存在乐观事务和悲观事务。如果使用乐观事务,并且冲突比较大,那么很可能 TiFlash 要花费大量时间在多余的行转列上。

KVStore 的落盘模式相关问题

理论上 KVStore 也可以做到独立写盘,从而使得 DM 的落盘进度不会阻塞 Raft Log 的回收。缺点是会使 KVStore 完全变成上游,写链路更长。虽然我们底层用的 PS,Compaction 相对较少,但同样有写放大。但这目前也无法实现,因为:

  1. KVStore 落盘是全量的,KVStore 和 DM 的内存操作又绑在一块。
    这导致在落盘 KVStore 前必须先落盘 DM。并且整个过程还需要加自己的锁,否则会导致数据丢失,而加锁导致阻塞 Apply。特别在一些场景下,少量的 Raft Log 就会导致 KVStore 和 DM 的落盘,严重影响读取延迟。
  2. Raftstore V1 的 Apply 落盘又是同步的。
    在 Raftstore V1 中,写入的数据可能在操作系统的 Page Cache 中,也有可能被刷入了磁盘。如果是前者,那么会在 raftlog_gc 等地方被显式地 sync。但困难在于,V1 中无法精确获得这些时刻,从而进行通知。又因为 TiFlash 的状态不能落后于 Proxy,否则 Proxy 的 applied_index 可能比 KVStore 新从而丢数据。所以这里索性当做同步落盘处理,让 TiFlash 先落盘。代价是我们要劫持 TiKV 所有可能写 apply state 的行为,哪怕这个写不是 sync 写。后面会介绍我的一些异步落盘的想法。

一个优化方案是解耦 KVStore 和 DM 的落盘。也就是在 DM 落盘后,再清理掉 KVStore 中的数据。这需要将 Region 中的数据拆分成 KV 对落盘,但这会失去对 KV 对做聚合的能力,从而将顺序写转换为随机写,如果写入很密集,性能也许会比较差,所以这个在功能和性能上都依赖 UniPS。

另一种方案比较简单,也就是限制由 KVStore,实际上就是 Raftstore 发起的落盘,改为由 DM 发起。但这个方案并不感知 Raft Log 的占用,可能导致它膨胀。

前面提到异步落盘 KVStore 的问题,一个思路是落盘时使用过去的状态+当前的数据。但存在一些问题:

  1. 这个“过去的状态”也需要比 DM 的落盘状态要新,所以还是要先加锁获取 KVStore 状态,再无锁落盘 DM,再用旧状态落盘 KVStore。这样不能解耦和 DM 的落盘,但能够在落盘 DM 的时候无锁已经很好了。
  2. Split/Merge 或者可能 Apply Snapshot 改变了全局状态。这样的指令在 V1 中是不能被重放的,不然新 Split 出来的 Region 可能和重启前已经被 Split 和 Persist 出来的 Region 冲突。这样就需要在处理这些 Admin 指令的时候同步等待异步的 Persist 完成。其实更简单的方式是根据之前加锁获取的状态来推断有没有执行这些 Admin。
  3. 需要让 KVStore 支持其他命令的重放。目前来看,应该存在一些 corner case。
  4. 需要让 KVStore 通知 Proxy,当前落盘的 applied_index 并不是期望的 applied_index。这实际上破坏了 TiKV 的 MultiRaft 约束,更好的方式是拒接来自 Proxy 的落盘请求,然后从 KVStore 重新主动发起一个。
  5. 落盘 KVStore 同样需要加锁,从而阻塞 Raft 层的写入。

另一种方案是过去的状态和过去的数据。比如可以在 KVStore 在落盘时,新开一个 Memtable 处理新写入。此时需要处理新 Memtable 上的 Write 可能依赖老 Memtable 上的 Default 之类的问题。这样的好处是在落盘 KVStore 的时候都不需要加锁了。但是还存在两个问题:

  1. 在这前面需要落盘 DM,当然这个锁先前说了可以去掉。
  2. 如果写入很大,那么可能在旧的 Memtable 还没写完之前,新的 Memtable 就满了。这样还是 Write Stall。

如果希望彻底和 DM 解耦,就需要想办法保存上次 DM 落盘到现在落盘 KVStore 期间被写到 DMCache 上的数据。这是困难的。

KVStore 如何处理事务

在每一次 Raftstore 的 apply 写入时,会遍历所有 write 写入,并进行事务提交,也就是将数据从 KVStore 移动到 DeltaMerge。事务提交并不一定落盘,大部分情况是写在 DeltaMerge 的 DeltaCache 中的。
如果出现事务 rollback 回滚,则 TiKV 不仅会删除掉之前写的 default 和 lock,还会写一条 Rollback 记录,它也会被写到 Write CF 中,其用途是避免同 start_ts 事务再次被发起,client 需要用新请求的 start_ts。
可以看到,因为共识层的存在,TiFlash 无需处理事务 rollback 的问题。这也是 KVStore 存在的意义之一。

KVStore 的存储格式

是否直接用 protobuf 存储 Region?

protobuf 具有的几个特性让它不适合存储 Region:

  1. 较大的 size 下性能较差
  2. 不能只读取部分数据

是否使用 flag 存储 Region Extension?

https://github.com/pingcap/tiflash/issues/8590 不建议这样做。

Raft 机制带来的内存和存储开销

有没有可能 TiFlash 自己 truncate 日志呢?理论上 Learner 不会成为 Leader 从而发送日志,也不会处理 Follower Snapshot 请求。而 Raft 协议本身就是让每个节点自己做 Snapshot 然后 truncate 日志的。

我们在云上 TiFlash 做这样的优化,因为云上使用的 UniPS 对内存更敏感。PageDirectory 为每个 Page 占用大约 0.5KB 的内存。另一方面,UniPS 全部受我们控制,所以相比 Raft Engine 也更好做透明的回收。透明回收小于 persisted applied_index 的所有 Entry,如果 Raftstore 会访问已经被回收的 Entry,会给一个 Panic。

为什么 TiFlash 使用 DeltaTree 作为存储

目的是为了适应频繁的更新。我们采用类似 CStore的思路,引入了 PageStorage 这个对象存储。其中针对写优化的部分称为 Delta 层,类似于 RocksDB 的 L0,存储在 PageStorage 中。针对读优化的部分称为 Stable 层,以 DTFile 文件的形式存储,但文件路径在 PageStorage 作为 External Page 的形式维护。

存储模型的进一步讨论

和 StarRocks 的比较

例如可以将 update 操作分为 delete 和 insert 操作。查询时,同时查询 delete 和 insert,并决定最终的输出。StarRocks 使用这样的方式,他们指出 Delete+Insert 这样的模式有利于下推 Filter。StarRocks 据此实现了主键模型
这里需要区分他们的更新模型,也就是一种不支持 MVCC,始终返回最新数据的模型。这种模型应该就是一种类似 LSM 的方案,在 Compaction 的时候只保留一个版本。但是在查询的时候仍然需要 merge 多个版本,并且不支持下推 filter。
主键模型的优势就是查询时不需要 merge,并且支持下推 filter 和索引。这种方式主要是将主键索引加载到内存中,对于 Update 操作,通过主键索引找到记录的位置,写一个 Delete,然后再写一个 Insert。可以发现这种方案仍然是不支持 MVCC 的,我理解如果要支持 MVCC 那么 merge 可能是必然的。
此外,主键模型对内存是有开销的,我理解这个应该不是关键问题。首先,如果数据有冷热之分,可以持久化一部分主键索引到磁盘上。其次,这个场景在大宽表有优势。

来自 TiKV 的约束

从 Raft 层接入数据导致 TiFlash 的存储层的分区会收到 TiKV Key Format 的影响。例如尽管 TiFlash 的 Segment 和 TiKV 的 Region 并不对应,Segment 远大于 Region。但它们都被映射到同一个 Key Range 上。

这就导致 TiFlash 数据的物理排列一定是根据 TiKV 的主键有序的,TiFlash 无法自行指定主键。另外 TiFlash 本身也没有二级索引。

目前来自 TiKV 的约束有:

  1. MVCC 字段
    如果要和 TiDB 一起玩,就必须要支持 MVCC,不能只保存最新的版本。
  2. Unique 的主键

DM 的 Delta 层是如何实现的?

PageStorage 先前使用 Append 写加上 GC 的方案,但带来写放大、读放大和空间放大。因为这里 GC 采用的 Copy Out 的方式,所以理论上写放大和空间放大构成一个 trade off:

  1. 如果允许更少的有效数据和更多的碎片,那么空间放大更严重
  2. 否则,写放大更严重

旧的 PageStorage 主要存在下面的问题:

  1. GC 开销很大,因为需要遍历所有的 Version 或者说 Snapshot 才能得到可以被安全删除的数据。这样会产生很多额外的遍历。
  2. 每张表一个实例,如果存在很多小表,则会产生非常多的文件,甚至会用光 fd。
  3. 冷热数据分离。因为 meta 一般会被频繁更新,而实际上存在一些比较冷的 data。这会导致冷 data 阻碍 meta 进行 gc,这样会产生空间放大。到一定程度之后,又会触发 gc,进一步加剧问题。

在 SSD 盘上,随机写和顺序写的差距不大,原因是 FTL 会将随机写转换为顺序写,所以寻址相关的开销并不是很大。尽管如此,顺序写依然存在优势,首先顺序写可以做聚合,同样的 IOPS 写入带宽是会比随机写要大很多,然后是顺序写的 gc 会更容易。此外,因为变成随机读,性能会变差。特别是对类似 Raft Log 这样的 scan 场景。

新一版本的设计,TiFlash 会通过 SpaceMap 尽量选择从已有的文件中分配一块合适的空间用来写入 blob。当 blob 被分配完毕后,多个 writer 可以并发地写自己的部分。在写入 blob 完成后,会写 WAL 记录相关元信息。在这之后就可以更新内存中的数据。

为什么 DM 的 Stable 只有一层?

DM 的设计目标包含优化读性能和支持 MVCC 过滤。这就导致要解决下面的场景

TiFlash 有比较多的数据更新操作,与此同时承载的读请求,都会需要通过 MVCC 版本过滤出需要读的数据。而以 LSM Tree 形式组织数据的话,在处理 Scan 操作的时候,会需要从 L0 的所有文件,以及其他层中与查询的 key-range 有 overlap 的所有文件,以堆排序的形式合并、过滤数据。在合并数据的这个入堆、出堆的过程中 CPU 的分支经常会 miss,cache 命中也会很低。测试结果表明,在处理 Scan 请求的时候,大量的 CPU 都消耗在这个堆排序的过程中。

另外,采用 LSM Tree 结构,对于过期数据的清理,通常在 level compaction 的过程中,才能被清理掉(即 Lk-1 层与 Lk 层 overlap 的文件进行 compaction)。而 level compaction 的过程造成的写放大会比较严重。当后台 compaction 流量比较大的时候,会影响到前台的写入和数据读取的性能,造成性能不稳定。

为了缓解单层带来的写放大,DM 按照 key range 分成了多个 Segment。每个 Segment 中包含自己的 Stable 和 Delta。其中 Delta 合并 Stable 会产生一个新的 Stable。

为什么 TiFlash 按 TSO 升序存储?

TiKV 的 TSO 按照逆序存,有利于找新版本。
TiFlash 因为都是处理扫表,所以逆序的收益不是很大。ClickHouse 使用升序存储,所以 TiFlash 也沿用了升序。
但这里就导致在处理 Snapshot 写入的时候,需要读完每个 row key 的所有版本,并在一个 read 调用中返回给下游的 stream。

TiFlash 如何处理 Raft Snapshot?

为什么 TiFlash 不处理 DeleteRange?

TiKV 通过 DeleteRange 来删表。TiFlash 则是通过拉取 DDL,并确保已经过了 gc safepoint 后,才会物理删除表。

需要注意的是,除了删表之外,pd 可能从 TiFlash 调度走某个 Region,这也涉及删除操作。对于这样的操作,TiFlash 就得立即响应。

在 gc 时,在 write cf 上写一个 DEL 记录,也就是所谓的 tombstone key 是比较少见的。现在的做法是在 Compaction 的时候将这些 key filter 掉。当然在提交事务的时候,DEL lock cf 是很常见的。

读取

为什么 TiFlash 没有 buffer pool

对于 AP 负载,扫表的数据规模很大,Cache 起不到太大作用。

资源管理

弹性的资源管理和存算分离

在目前的计算机架构下,进程是资源的分配单位。这就意味着如果程序对除了 CPU 之外的某个资源的需求存在很大的弹性,那么就需要将这一部分单独剥离出来。
TiFlash Cloud 中就使用了存算分离,当然还使用了 OSS 等方案,但我认为是正交的设计。

内存

历史上计算层出现过不少因为查询过大导致的 OOM,计算层通过 kill query 或者 spill 的方式进行解决。但存储层目前还缺少这块。理论上存储层的开销主要分为几类:

  1. Memtable
    包含 KVStore 的 RegionData 和 DeltaTree 的 DeltaCache。
    这类场景下,OOM 主要发生在大事务场景。
  2. Cache
    主要用来服务计算节点,列存主要是扫表,所以没有做 Block cache 或者 row cache。
  3. 索引
    包含 DeltaTree 的 DeltaIndex,PageStorage 的 PageDirectory 等。
  4. Compact 相关,比如 delte merge 等
  5. 行转列相关

在一些场景下,因为存储层和计算层并不互相感知,会导致存储层会被计算层的大任务干到 OOM 或者报异常。而实际上这些任务可以被 kill,stall 或者通过 kill query 抢占计算层的内存。

因此,在 TiFlash 侧实现一个统一的内存管理还是有必要的。

空指针

严格来讲避免空指针也不完全算是内存管理。但确实是工作中遇到的一个比较关键的问题。我在 分布式架构和高并发相关场景 这篇文章里面说吧。

线程

IO

CPU

TiFlash Cloud

快速扩容(FAP) What & why?

目的:

  1. 复用 TiFlash 行转列的结果。减少 TiKV 生成、传输和 TiFlash 接收、转换 Snapshot 的开销。
    在测试中,发现能够减少 96% 的 CPU 开销和 20% 的内存开销。
    如果提升调度的 limiter,能够大幅提高吞吐量,体现为添加副本总时间的减少。但该增长不是线性的,也取决于 TiFlash 侧线程池的大小,以及串行 ingest 的开销。
    需要注意,因为 Region 和 Raft Group 绑定,导致 FAP 必须等待 apply Confchange 之后的 Checkpoint,所以对于单个小 Region 来说,可能要花费更长的时间来处理。
  2. 利用如 S3 的特性,减少跨 Region 通信。
  3. 提高副本迁移,特别是单副本迁移的效率。
  4. 在扩容场景下,新节点可能因为处理全量 Snapshot 更慢,导致进度落后,从而进一步触发全量 Snapshot。此时新机器无法处理被 dispatch 过来的请求。

要点:

  1. 使用 PageStorage 替换 RaftEngine。这样使得 Raft、KVStore 和 DeltaTree 数据都一起被存到同一个 checkpoint 里面,保证原子性和一致性。
  2. 副本选择和由 Learner 管理的副本创建。用来快速扩容的 TiFlash Checkpoint,必须要比扩容对应的 confchange log entry 要新。这是因为 TiKV 通过一个 Snapshot 来帮助新 node 追日志,而这个 Snapshot 必然在 confchange 后产生。如果接受一个更早的 Checkpoint,那么就要确保 raft 能够给新 peer 发送 confchange 前的日志。即使能,这也意味着新 peer 要处理添加自己的 confchange cmd。即使通过忽略等方案处理,那么在这之前的 batch split cmd 就需要伪装成生成 Checkpoint 的那个 peer,并将这个 region 重新切开(涉及一些行转列和写盘)。而如果与此同时,batch split 得到的某个 split 的最新版本又通过正常途径调度过来,并且在 apply snapshot,那么这里就可能产生 region overlap 导致的数据问题。可以看出,因为违反了 TiKV 的约束,所以产生了很多的潜在问题。
  3. 注入数据。需要注意,原有的 TiKV 的通过 Snapshot 初始化副本的流程需要重新走一遍。
  4. 对旧版本数据的清理。

这个 feature 类似于 Learner Snapshot,所以为什么不通过 Follower/Learner Snapshot 来实现 FAP 呢?

  1. TiKV 主要需要该 Feature 来避免跨地区的 Snapshot 复制,而 TiFlash 需要该 Feature 实现异构的 Snapshot,侧重点上有所不同。
  2. 该 feature 需要在 TiKV 或者 PD 等组件中实现一定的调度机制。所以 FAP 实际可以视为一个部分的实现,后续有可能进行推广。届时 FAP 的 phase 1 过程就有可能被移动到 prehandle snapshot 中处理了。
  3. Follower Snapshot 有可能会失败,例如 Follower 节点实际上做不了该 Snapshot。此时 Snapshot 依然会由 Leader 来处理。目前 TiKV 的模型还不支持这种模式。

使用 UniPS 替换 RaftEngine

目前 TiKV 使用 engine_traits 描述了一个可以用来作为 raftstore 的存储的 engine 所需要的接口。这些接口基本是基于 RocksDB 而抽象出来的。因此 UniPS 需要模拟出其中关键的特性,例如 WriteBatch 等。

UniPS 的性能劣于 RaftEngine,写入延迟大约是两倍。但是仍有不少优化空间。

为什么 TiFlash Cloud 目前还是两副本?

目前快速恢复还是实验状态。TiFlash 重启后也需要进行一些整理和追日志才能服务,可能影响 HA,这些需要时间优化。尽管如此,快速恢复依然是一个很好的特性,因为:

  1. 快速恢复在 1 wn 下,可以从本节点重启,减少 TiKV 生成 Snapshot 的负担。而这个负担在 v1 版本的 TiKV 上是比较大的。
  2. 减少宕机一个节点恢复后,集群恢复到正常 2 副本的时间。

因为基于 Raft,所以本地数据的丢失只会导致从上一个 S3 Checkpoint 开始回放。如果只有一个存储节点,会失去 HA 特性。

S3 在 TiFlash Cloud 中起到什么作用?

  1. TiFlash Cloud 会定期上传 Checkpoint 到 S3 上,Checkpoint 是一个完整的快照,可以用来做容灾。即使在存储节点宕机后,其上传的那部分数据依然可以被用来查询,可能只能用来服务 stale read?
  2. TiFlash 计算节点可以从 S3 获得数据,相比从存储节点直接获取要更为便宜。存储节点只需要提供一些比较新的数据的读取,减少压力。
  3. 快速扩容逻辑可以复用其他存储节点的数据,此时新节点并不需要从 TiKV 或者其他 TiFlash 获得全部的数据。副本迁移同理,不需要涉及全部数据的移动。

尽管如此,S3 并不是当前 TiFlash 数据的全集。本地会存在:

  1. 上传间隔时间内,还没有上传到 S3 的数据。
  2. 因为生命周期太短,在上传前就被 tombstone 的数据。
  3. 尚在内存中的数据。

S3 vs EBS

对于 S3 而言:

  1. 具备 99.999999999% 的持久性和 99.99% 的可用性。
    也就是说一天中的不可用时间大约在 9s 左右。

定价

  1. PUT/POST/LIST/COPY 0.005
  2. GET/SELECT 0.0004
  3. 存储每 GB 0.022 USD 每月

可以看到,S3 的定价相比 EBS 要便宜不少。此外,从灾备上来讲,使用 EBS 可能需要为跨 AZ 容灾付出更多的成本,而 S3 可以实现跨 AZ 容灾。

当然 S3 也有缺陷,比如访问延迟比较高。

Reference

  1. https://docs.pingcap.com/zh/tidb/stable/troubleshoot-hot-spot-issues
  2. https://www.infoq.com/articles/raft-engine-tikv-database/ RaftEngine
  3. https://www.zhihu.com/question/47544675 固态硬盘性能
  4. https://docs.pingcap.com/zh/tidb/stable/titan-overview Titan 设计
  5. Fast scans on key-value stores