etcd raft library设计原理和使用

早在2013年11月份,在raft论文还只能在网上下载到草稿版时,我曾经写过一篇blog对其进行简要分析。4年过去了,各种raft协议的讲解铺天盖地,raft也确实得到了广泛的应用。其中最知名的应用莫过于etcd。etcd将raft协议本身实现为一个library,位于https://github.com/coreos/etcd/tree/master/raft,然后本身作为一个应用使用它。

本文不讲解raft协议核心内容,而是站在一个etcd raft library使用者的角度,讲解要用上这个library需要了解的东西。

这个library使用起来相对来说还是有点麻烦。官方有一个使用示例在 https://github.com/coreos/etcd/tree/master/contrib/raftexample。整体来说,这个库实现了raft协议核心的内容,比如append log的逻辑,选主逻辑,snapshot,成员变更等逻辑。需要明确的是:library没有实现消息的网络传输和接收,库只会把一些待发送的消息保存在内存中,用户自定义的网络传输层取出消息并发送出去,并且在网络接收端,需要调一个library的函数,用于将收到的消息传入library,后面会详细说明。同时,library定义了一个Storage接口,需要library的使用者自行实现。

Storage接口如下:

// Storage is an interface that may be implemented by the application// to retrieve log entries from storage.//// If any Storage method returns an error, the raft instance will// become inoperable and refuse to participate in elections; the// application is responsible for cleanup and recovery in this case.type Storage interface {    // InitialState returns the saved HardState and ConfState information.
    InitialState() (pb.HardState, pb.ConfState, error)    // Entries returns a slice of log entries in the range [lo,hi).
    // MaxSize limits the total size of the log entries returned, but
    // Entries returns at least one entry if any.
    Entries(lo, hi, maxSize uint64) ([]pb.Entry, error)    // Term returns the term of entry i, which must be in the range
    // [FirstIndex()-1, LastIndex()]. The term of the entry before
    // FirstIndex is retained for matching purposes even though the
    // rest of that entry may not be available.
    Term(i uint64) (uint64, error)    // LastIndex returns the index of the last entry in the log.
    LastIndex() (uint64, error)    // FirstIndex returns the index of the first log entry that is
    // possibly available via Entries (older entries have been incorporated
    // into the latest Snapshot; if storage only contains the dummy entry the
    // first log entry is not available).
    FirstIndex() (uint64, error)    // Snapshot returns the most recent snapshot.
    // If snapshot is temporarily unavailable, it should return ErrSnapshotTemporarilyUnavailable,
    // so raft state machine could know that Storage needs some time to prepare
    // snapshot and call Snapshot later.
    Snapshot() (pb.Snapshot, error)
}

这些接口在library中会被用到。熟悉raft协议的人不难理解。上面提到的官方示例https://github.com/coreos/etcd/tree/master/contrib/raftexample中使用了library自带的MemoryStorage,和etcd的wal和snap包做持久化,重启的时候从wal和snap中获取日志恢复MemoryStorage。

要提供这种IO/网络密集型的东西,提高吞吐最好的手段就是batch加批处理了。etcd raft library正是这么做的。

下面看一下为了做这事,etcd提供的核心抽象Ready结构体:

// Ready encapsulates the entries and messages that are ready to read,// be saved to stable storage, committed or sent to other peers.// All fields in Ready are read-only.type Ready struct {    // The current volatile state of a Node.
    // SoftState will be nil if there is no update.
    // It is not required to consume or store SoftState.
    *SoftState    // The current state of a Node to be saved to stable storage BEFORE
    // Messages are sent.
    // HardState will be equal to empty state if there is no update.
    pb.HardState    // ReadStates can be used for node to serve linearizable read requests locally
    // when its applied index is greater than the index in ReadState.
    // Note that the readState will be returned when raft receives msgReadIndex.
    // The returned is only valid for the request that requested to read.
    ReadStates []ReadState    // Entries specifies entries to be saved to stable storage BEFORE
    // Messages are sent.
    Entries []pb.Entry    // Snapshot specifies the snapshot to be saved to stable storage.
    Snapshot pb.Snapshot    // CommittedEntries specifies entries to be committed to a
    // store/state-machine. These have previously been committed to stable
    // store.
    CommittedEntries []pb.Entry    // Messages specifies outbound messages to be sent AFTER Entries are
    // committed to stable storage.
    // If it contains a MsgSnap message, the application MUST report back to raft
    // when the snapshot has been received or has failed by calling ReportSnapshot.
    Messages []pb.Message    // MustSync indicates whether the HardState and Entries must be synchronously
    // written to disk or if an asynchronous write is permissible.
    MustSync bool}

可以说,这个Ready结构体封装了一批更新,这些更新包括:

  • pb.HardState: 包含当前节点见过的最大的term,以及在这个term给谁投过票,已经当前节点知道的commit index
  • Messages: 需要广播给所有peers的消息
  • CommittedEntries:已经commit了,还没有apply到状态机的日志
  • Snapshot:需要持久化的快照

库的使用者从node结构体提供的一个ready channel中不断的pop出一个个的Ready进行处理,库使用者通过如下方法拿到Ready channel:

func (n *node) Ready() <-chan Ready { return n.readyc }

应用需要对Ready的处理包括:

  1. 将HardState, Entries, Snapshot持久化到storage。
  2. 将Messages(上文提到的msgs)非阻塞的广播给其他peers
  3. 将CommittedEntries(已经commit还没有apply)应用到状态机。
  4. 如果发现CommittedEntries中有成员变更类型的entry,调用node的ApplyConfChange()方法让node知道(这里和raft论文不一样,论文中只要节点收到了成员变更日志就应用)
  5. 调用Node.Advance()告诉raft node,这批状态更新处理完了,状态已经演进了,可以给我下一批Ready让我处理。

应用通过raft.StartNode()来启动raft中的一个副本,函数内部通过启动一个goroutine运行

func (n *node) run(r *raft)

来启动服务。

应用通过调用

func (n *node) Propose(ctx context.Context, data []byte) error

来Propose一个请求给raft,被raft开始处理后返回。

增删节点通过调用

func (n *node) ProposeConfChange(ctx context.Context, cc pb.ConfChange) error

node结构体包含几个重要的channel:

// node is the canonical implementation of the Node interfacetype node struct {
    propc      chan pb.Message
    recvc      chan pb.Message
    confc      chan pb.ConfChange
    confstatec chan pb.ConfState
    readyc     chan Ready
    advancec   chan struct{}
    tickc      chan struct{}
    done       chan struct{}
    stop       chan struct{}
    status     chan chan Status

    logger Logger
}
  • propc: propc是一个没有buffer的channel,应用通过Propose接口写入的请求被封装成Message被push到propc中,node的run方法从propc中pop出Message,append自己的raft log中,并且将Message放入mailbox中(raft结构体中的msgs []pb.Message),这个msgs会被封装在Ready中,被应用从readyc中取出来,然后通过应用自定义的transport发送出去。
  • recvc: 应用自定义的transport在收到Message后需要调用
    func (n *node) Step(ctx context.Context, m pb.Message) error

    来把Message放入recvc中,经过一些处理后,同样,会把需要发送的Message放入到对应peers的mailbox中。后续通过自定义transport发送出去。

  • readyc/advancec: readyc和advancec都是没有buffer的channel,node.run()内部把相关的一些状态更新打包成Ready结构体(其中一种状态就是上面提到的msgs)放入readyc中。应用从readyc中pop出Ready中,对相应的状态进行处理,处理完成后,调用
    rc.node.Advance()

    往advancec中push一个空结构体告诉raft,已经对这批Ready包含的状态进行了相应的处理,node.run()内部从advancec中得到通知后,对内部一些状态进行处理,比如把已经持久化到storage中的entries从内存(对应type unstable struct)中删除等。

  • tickc:应用定期往tickc中push空结构体,node.run()会调用tick()函数,对于leader来说,tick()会给其他peers发心跳,对于follower来说,会检查是否需要发起选主操作。
  • confc/confstatec:应用从Ready中拿出CommittedEntries,检查其如果含有成员变更类型的日志,则需要调用
    func (n *node) ApplyConfChange(cc pb.ConfChange) *pb.ConfState

    这个函数会push ConfChange到confc中,confc同样是个无buffer的channel,node.run()内部会从confc中拿出ConfChange,然后进行真正的增减peers操作,之后将最新的成员组push到confstatec中,而ApplyConfChange函数从confstatec pop出最新的成员组返回给应用。

时间: 2024-08-28 17:51:49

etcd raft library设计原理和使用的相关文章

kafka入门:简介、使用场景、设计原理、主要配置及集群搭建(转)

问题导读: 1.zookeeper在kafka的作用是什么? 2.kafka中几乎不允许对消息进行"随机读写"的原因是什么? 3.kafka集群consumer和producer状态信息是如何保存的? 4.partitions设计的目的的根本原因是什么? 一.入门 1.简介 Kafka is a distributed,partitioned,replicated commit logservice.它提供了类似于JMS的特性,但是在设计实现上完全不同,此外它并不是JMS规范的实现.k

Atitit.ioc&#160;动态配置文件guice&#160;设计原理

Atitit.ioc 动态配置文件guice 设计原理 1. Bat启动时注入配置文件1 2. ioc调用1 3. Ioc 分发器 配合 apche  MethodUtils.invokeStaticMethod2 1. Bat启动时注入配置文件 SET JAVA_HOME=C:\Program Files\Java\jdk1.8.0_71 set  RESIN-HOME=c:\resin-4.0.22 set classpath=%classpath%;%RESIN-HOME%\lib\jas

BigPipe设计原理

高性能页面加载技术--BigPipe设计原理及Java简单实现 1.技术背景 动态web网站的历史可以追溯到万维网初期,相比于静态网站,动态网站提供了强大的可交互功能.经过几十年的发展,动态网站在互动性和页面显示效果上有了很大的提升,但是对于网站动态网站的整体页面加载架构没有做太大的改变.对于用户而言,页面的加载速度极大的影响着用户体验感.与静态网站不同,除了页面的传输加载时间外,动态网站还需考虑服务端数据的处理时间.像facebook这样大型的用户社交网站,必须考虑用户访问速度问题, 传统we

Atitit.异常的设计原理与&#160;策略处理&#160;java&#160;最佳实践&#160;p93

Atitit.异常的设计原理与 策略处理 java 最佳实践 p93 1 异常方面的使用准则,答案是::2 1.1 普通项目优先使用异常取代返回值,如果开发类库方面的项目,最好异常机制与返回值都提供,由调用者决定使用哪种方式..2 1.2 优先把异常抛出到上层处理..异常本身就是为了方便把异常情况抛出到上层处理..2 1.3 对于 HYPERLINK \l _Toc6222 方法调用结果异常情况返回策略,最终会有四种策略状况,2 1.4 返回null  还是异常??2 2 异常的由来与设计3 2

Atitit ati&#160;licenseService &#160;&#160;&#160;设计原理

Atitit ati licenseService    设计原理 C:\0workspace\AtiPlatf\src_atibrow\com\attilax\license\LicenseX.java V1 更具时间超是 V2   按照时间慢的百分率.. V3 草案.. Laicense file ,hto sh aes time.. Invoke System.out.println( licenseX.isCanUse_byUsePercent("2016-05-01") );

以属性为核心驱动的 全领域通用架构设计原理 (简称:属性架构原理)

以属性为核心驱动的全领域通用架构设计原理 (简称:属性架构原理) 联系方式:13547930387 Email:[email protected] 一.个人声明 我,参加工作也有5年多了,是一名普通的不能在普通的程序员,一直在使用公司自己的产品进行开发,因此技术比较菜,此设计完全是按照自己天真的想法而设计的,如果有不合理或很搞笑的地方,请轻拍,由衷的希望大家能提出宝贵的意见: 根据此设计原理我也做了一个简单的(demo)架构来支撑和验证此理论的可行性,由于技术功底不太好,有不合理之处请大家谅解,

深入理解kafka设计原理

最近开研究kafka,下面分享一下kafka的设计原理.kafka的设计初衷是希望作为一个统一的信息收集平台,能够实时的收集反馈信息,并需要能够支撑较大的数据量,且具备良好的容错能力. 1.持久性 kafka使用文件存储消息,这就直接决定kafka在性能上严重依赖文件系统的本身特性.且无论任何OS下,对文件系统本身的优化几乎没有可能.文件缓存/直接内存映射等是常用的手段.因为kafka是对日志文件进行append操作,因此磁盘检索的开支是较小的;同时为了减少磁盘写入的次数,broker会将消息暂

BeanFactory容器的设计原理

XmlBeanFactory设计的类继承关系 1.BeanFactory接口提供了使用IoC容器的规范.在这个基础上,Spring还提供了符合这个IoC容器接口的一系列容器的实现供开发人员使用. 2.我们以XmlBeanFactory的实现为例来说明简单IoC容器的设计原理. 3.可以看到,作为一个简单IoC容器系列最底层实现的XmlBeanFactory,与我们在Spring应用中用到的上下文相比,有一个非常明显的特点:它只提供最基本的IoC容器的功能. 4.理解这一点有助于我们理解Appli

Scala函数式编程设计原理 第一课 编程范式(Programming Paradigms)

我使用Scala有一两年的时间了,这门语言仿佛有一种魔力,让人用过就不想放手.Scala给我的整个程序生涯带来了非常深刻的影响,让我学会了函数式编程,让我知道了世界上居然还有这么一种优雅.高效.强大的语言. Scala在国外已经非常流行,但是不知为何,在国内总是不温不火,在此,我特别想为Scala这门语言在国内的发展做一些事情.不才不敢谈Scala的编程经验,因为要成为Scala大神还有很长的路要走,只好翻译一份Scala视频教程以飨读者,大家发现有误的地方,请多多批评指教. 这个视频的作者是S