mirror of
https://github.com/EchoVault/SugarDB.git
synced 2025-10-19 22:34:40 +08:00
Refactored broadcast for joining raft cluster. Instead of broadcasting repeatedly with a ticker, the message will be relayed using gossip protocol until it reaches the raft leader.
Created scaffolding for forwarding mutation commands from replica nodes to leader node.
This commit is contained in:
@@ -27,3 +27,4 @@ CMD "./server" \
|
|||||||
"--aclConfig=${ACL_CONFIG}" \
|
"--aclConfig=${ACL_CONFIG}" \
|
||||||
"--requirePass=${REQUIRE_PASS}" \
|
"--requirePass=${REQUIRE_PASS}" \
|
||||||
"--password=${PASSWORD}" \
|
"--password=${PASSWORD}" \
|
||||||
|
"--forwardCommand=${FORWARD_COMMAND}" \
|
11
src/main.go
11
src/main.go
@@ -254,11 +254,13 @@ func (server *Server) handleConnection(ctx context.Context, conn net.Conn) {
|
|||||||
|
|
||||||
connRW.Write(r.Response)
|
connRW.Write(r.Response)
|
||||||
connRW.Flush()
|
connRW.Flush()
|
||||||
|
} else if server.config.ForwardCommand {
|
||||||
// TODO: Add command to AOF
|
// Forward message to leader and return immediate OK response
|
||||||
|
server.memberList.ForwardDataMutation(ctx, message)
|
||||||
|
connRW.Write([]byte(utils.OK_RESPONSE))
|
||||||
|
connRW.Flush()
|
||||||
} else {
|
} else {
|
||||||
// TODO: Forward message to leader and wait for a response
|
connRW.Write([]byte("-Error not cluster leader, cannot carry out command\r\n\r\n"))
|
||||||
connRW.Write([]byte("-Error not cluster leader, cannot carry out command\r\n\n"))
|
|
||||||
connRW.Flush()
|
connRW.Flush()
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
@@ -378,6 +380,7 @@ func (server *Server) Start(ctx context.Context) {
|
|||||||
HasJoinedCluster: server.raft.HasJoinedCluster,
|
HasJoinedCluster: server.raft.HasJoinedCluster,
|
||||||
AddVoter: server.raft.AddVoter,
|
AddVoter: server.raft.AddVoter,
|
||||||
RemoveRaftServer: server.raft.RemoveServer,
|
RemoveRaftServer: server.raft.RemoveServer,
|
||||||
|
IsRaftLeader: server.raft.IsRaftLeader,
|
||||||
})
|
})
|
||||||
server.raft.RaftInit(ctx)
|
server.raft.RaftInit(ctx)
|
||||||
server.memberList.MemberListInit(ctx)
|
server.memberList.MemberListInit(ctx)
|
||||||
|
@@ -8,23 +8,27 @@ import (
|
|||||||
|
|
||||||
type BroadcastMessage struct {
|
type BroadcastMessage struct {
|
||||||
NodeMeta
|
NodeMeta
|
||||||
Action string `json:"Action"`
|
Action string `json:"Action"`
|
||||||
Content string `json:"Content"`
|
Content string `json:"Content"`
|
||||||
|
ContentHash string `json:"ContentHash"`
|
||||||
}
|
}
|
||||||
|
|
||||||
// Invalidates Implements Broadcast interface
|
// Invalidates Implements Broadcast interface
|
||||||
func (broadcastMessage *BroadcastMessage) Invalidates(other memberlist.Broadcast) bool {
|
func (broadcastMessage *BroadcastMessage) Invalidates(other memberlist.Broadcast) bool {
|
||||||
mb, ok := other.(*BroadcastMessage)
|
otherBroadcast, ok := other.(*BroadcastMessage)
|
||||||
|
|
||||||
if !ok {
|
if !ok {
|
||||||
return false
|
return false
|
||||||
}
|
}
|
||||||
|
|
||||||
if mb.ServerID == broadcastMessage.ServerID && mb.Action == "RaftJoin" {
|
switch broadcastMessage.Action {
|
||||||
return true
|
case "RaftJoin":
|
||||||
|
return broadcastMessage.Action == otherBroadcast.Action && broadcastMessage.ServerID == otherBroadcast.ServerID
|
||||||
|
case "MutateData":
|
||||||
|
return broadcastMessage.Action == otherBroadcast.Action && broadcastMessage.ContentHash == otherBroadcast.ContentHash
|
||||||
|
default:
|
||||||
|
return false
|
||||||
}
|
}
|
||||||
|
|
||||||
return false
|
|
||||||
}
|
}
|
||||||
|
|
||||||
// Message Implements Broadcast interface
|
// Message Implements Broadcast interface
|
||||||
|
@@ -17,6 +17,7 @@ type DelegateOpts struct {
|
|||||||
config utils.Config
|
config utils.Config
|
||||||
broadcastQueue *memberlist.TransmitLimitedQueue
|
broadcastQueue *memberlist.TransmitLimitedQueue
|
||||||
addVoter func(id raft.ServerID, address raft.ServerAddress, prevIndex uint64, timeout time.Duration) error
|
addVoter func(id raft.ServerID, address raft.ServerAddress, prevIndex uint64, timeout time.Duration) error
|
||||||
|
isRaftLeader func() bool
|
||||||
}
|
}
|
||||||
|
|
||||||
func NewDelegate(opts DelegateOpts) *Delegate {
|
func NewDelegate(opts DelegateOpts) *Delegate {
|
||||||
@@ -54,11 +55,23 @@ func (delegate *Delegate) NotifyMsg(msgBytes []byte) {
|
|||||||
|
|
||||||
switch msg.Action {
|
switch msg.Action {
|
||||||
case "RaftJoin":
|
case "RaftJoin":
|
||||||
if err := delegate.options.addVoter(msg.NodeMeta.ServerID, msg.NodeMeta.RaftAddr, 0, 0); err != nil {
|
// If the current node is not the cluster leader, re-broadcast the message
|
||||||
|
if !delegate.options.isRaftLeader() {
|
||||||
|
delegate.options.broadcastQueue.QueueBroadcast(&msg)
|
||||||
|
return
|
||||||
|
}
|
||||||
|
err := delegate.options.addVoter(msg.NodeMeta.ServerID, msg.NodeMeta.RaftAddr, 0, 0)
|
||||||
|
if err != nil {
|
||||||
fmt.Println(err)
|
fmt.Println(err)
|
||||||
}
|
}
|
||||||
case "MutateData":
|
case "MutateData":
|
||||||
// Mutate the value at a given key
|
// If the current node is not a cluster leader, re-broadcast the message
|
||||||
|
if !delegate.options.isRaftLeader() {
|
||||||
|
delegate.options.broadcastQueue.QueueBroadcast(&msg)
|
||||||
|
return
|
||||||
|
}
|
||||||
|
// Current node is the cluster leader, handle the mutation
|
||||||
|
fmt.Println(msg)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@@ -11,9 +11,9 @@ type EventDelegate struct {
|
|||||||
}
|
}
|
||||||
|
|
||||||
type EventDelegateOpts struct {
|
type EventDelegateOpts struct {
|
||||||
IncrementNodes func()
|
incrementNodes func()
|
||||||
DecrementNodes func()
|
decrementNodes func()
|
||||||
RemoveRaftServer func(meta NodeMeta) error
|
removeRaftServer func(meta NodeMeta) error
|
||||||
}
|
}
|
||||||
|
|
||||||
func NewEventDelegate(opts EventDelegateOpts) *EventDelegate {
|
func NewEventDelegate(opts EventDelegateOpts) *EventDelegate {
|
||||||
@@ -24,12 +24,12 @@ func NewEventDelegate(opts EventDelegateOpts) *EventDelegate {
|
|||||||
|
|
||||||
// NotifyJoin implements EventDelegate interface
|
// NotifyJoin implements EventDelegate interface
|
||||||
func (eventDelegate *EventDelegate) NotifyJoin(node *memberlist.Node) {
|
func (eventDelegate *EventDelegate) NotifyJoin(node *memberlist.Node) {
|
||||||
eventDelegate.options.IncrementNodes()
|
eventDelegate.options.incrementNodes()
|
||||||
}
|
}
|
||||||
|
|
||||||
// NotifyLeave implements EventDelegate interface
|
// NotifyLeave implements EventDelegate interface
|
||||||
func (eventDelegate *EventDelegate) NotifyLeave(node *memberlist.Node) {
|
func (eventDelegate *EventDelegate) NotifyLeave(node *memberlist.Node) {
|
||||||
eventDelegate.options.DecrementNodes()
|
eventDelegate.options.decrementNodes()
|
||||||
|
|
||||||
var meta NodeMeta
|
var meta NodeMeta
|
||||||
|
|
||||||
@@ -40,7 +40,7 @@ func (eventDelegate *EventDelegate) NotifyLeave(node *memberlist.Node) {
|
|||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
err = eventDelegate.options.RemoveRaftServer(meta)
|
err = eventDelegate.options.removeRaftServer(meta)
|
||||||
|
|
||||||
if err != nil {
|
if err != nil {
|
||||||
fmt.Println(err)
|
fmt.Println(err)
|
||||||
|
@@ -23,6 +23,7 @@ type MemberlistOpts struct {
|
|||||||
HasJoinedCluster func() bool
|
HasJoinedCluster func() bool
|
||||||
AddVoter func(id raft.ServerID, address raft.ServerAddress, prevIndex uint64, timeout time.Duration) error
|
AddVoter func(id raft.ServerID, address raft.ServerAddress, prevIndex uint64, timeout time.Duration) error
|
||||||
RemoveRaftServer func(meta NodeMeta) error
|
RemoveRaftServer func(meta NodeMeta) error
|
||||||
|
IsRaftLeader func() bool
|
||||||
}
|
}
|
||||||
|
|
||||||
type MemberList struct {
|
type MemberList struct {
|
||||||
@@ -48,11 +49,12 @@ func (m *MemberList) MemberListInit(ctx context.Context) {
|
|||||||
config: m.options.Config,
|
config: m.options.Config,
|
||||||
broadcastQueue: m.broadcastQueue,
|
broadcastQueue: m.broadcastQueue,
|
||||||
addVoter: m.options.AddVoter,
|
addVoter: m.options.AddVoter,
|
||||||
|
isRaftLeader: m.options.IsRaftLeader,
|
||||||
})
|
})
|
||||||
cfg.Events = NewEventDelegate(EventDelegateOpts{
|
cfg.Events = NewEventDelegate(EventDelegateOpts{
|
||||||
IncrementNodes: func() { m.numOfNodes += 1 },
|
incrementNodes: func() { m.numOfNodes += 1 },
|
||||||
DecrementNodes: func() { m.numOfNodes -= 1 },
|
decrementNodes: func() { m.numOfNodes -= 1 },
|
||||||
RemoveRaftServer: m.options.RemoveRaftServer,
|
removeRaftServer: m.options.RemoveRaftServer,
|
||||||
})
|
})
|
||||||
|
|
||||||
m.broadcastQueue.RetransmitMult = 1
|
m.broadcastQueue.RetransmitMult = 1
|
||||||
@@ -82,31 +84,29 @@ func (m *MemberList) MemberListInit(ctx context.Context) {
|
|||||||
log.Fatal(err)
|
log.Fatal(err)
|
||||||
}
|
}
|
||||||
|
|
||||||
go m.broadcastRaftAddress(ctx)
|
m.broadcastRaftAddress(ctx)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
func (m *MemberList) broadcastRaftAddress(ctx context.Context) {
|
func (m *MemberList) broadcastRaftAddress(ctx context.Context) {
|
||||||
ticker := time.NewTicker(5 * time.Second)
|
msg := BroadcastMessage{
|
||||||
|
Action: "RaftJoin",
|
||||||
for {
|
NodeMeta: NodeMeta{
|
||||||
msg := BroadcastMessage{
|
ServerID: raft.ServerID(m.options.Config.ServerID),
|
||||||
Action: "RaftJoin",
|
RaftAddr: raft.ServerAddress(fmt.Sprintf("%s:%d",
|
||||||
NodeMeta: NodeMeta{
|
m.options.Config.BindAddr, m.options.Config.RaftBindPort)),
|
||||||
ServerID: raft.ServerID(m.options.Config.ServerID),
|
},
|
||||||
RaftAddr: raft.ServerAddress(fmt.Sprintf("%s:%d",
|
|
||||||
m.options.Config.BindAddr, m.options.Config.RaftBindPort)),
|
|
||||||
},
|
|
||||||
}
|
|
||||||
|
|
||||||
if m.options.HasJoinedCluster() {
|
|
||||||
return
|
|
||||||
}
|
|
||||||
|
|
||||||
m.broadcastQueue.QueueBroadcast(&msg)
|
|
||||||
|
|
||||||
<-ticker.C
|
|
||||||
}
|
}
|
||||||
|
m.broadcastQueue.QueueBroadcast(&msg)
|
||||||
|
}
|
||||||
|
|
||||||
|
func (m *MemberList) ForwardDataMutation(ctx context.Context, cmd string) {
|
||||||
|
// This function is only called by non-leaders
|
||||||
|
// It uses the broadcast queue to forward a data mutation within the cluster
|
||||||
|
m.broadcastQueue.QueueBroadcast(&BroadcastMessage{
|
||||||
|
Action: "MutateData",
|
||||||
|
Content: cmd,
|
||||||
|
})
|
||||||
}
|
}
|
||||||
|
|
||||||
func (m *MemberList) MemberListShutdown(ctx context.Context) {
|
func (m *MemberList) MemberListShutdown(ctx context.Context) {
|
||||||
|
@@ -153,24 +153,23 @@ func (r *Raft) AddVoter(
|
|||||||
prevIndex uint64,
|
prevIndex uint64,
|
||||||
timeout time.Duration,
|
timeout time.Duration,
|
||||||
) error {
|
) error {
|
||||||
if !r.IsRaftLeader() {
|
if r.IsRaftLeader() {
|
||||||
return errors.New("not leader, cannot add voter")
|
raftConfig := r.raft.GetConfiguration()
|
||||||
}
|
if err := raftConfig.Error(); err != nil {
|
||||||
raftConfig := r.raft.GetConfiguration()
|
return errors.New("could not retrieve raft config")
|
||||||
if err := raftConfig.Error(); err != nil {
|
|
||||||
return errors.New("could not retrieve raft config")
|
|
||||||
}
|
|
||||||
|
|
||||||
for _, s := range raftConfig.Configuration().Servers {
|
|
||||||
// Check if a server already exists with the current attributes
|
|
||||||
if s.ID == id && s.Address == address {
|
|
||||||
return fmt.Errorf("server with id %s and address %s already exists", id, address)
|
|
||||||
}
|
}
|
||||||
}
|
|
||||||
|
|
||||||
err := r.raft.AddVoter(id, address, prevIndex, timeout).Error()
|
for _, s := range raftConfig.Configuration().Servers {
|
||||||
if err != nil {
|
// Check if a server already exists with the current attributes
|
||||||
return err
|
if s.ID == id && s.Address == address {
|
||||||
|
return fmt.Errorf("server with id %s and address %s already exists", id, address)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
err := r.raft.AddVoter(id, address, prevIndex, timeout).Error()
|
||||||
|
if err != nil {
|
||||||
|
return err
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
return nil
|
return nil
|
||||||
|
@@ -26,6 +26,7 @@ type Config struct {
|
|||||||
DataDir string `json:"dataDir" yaml:"dataDir"`
|
DataDir string `json:"dataDir" yaml:"dataDir"`
|
||||||
BootstrapCluster bool `json:"BootstrapCluster" yaml:"bootstrapCluster"`
|
BootstrapCluster bool `json:"BootstrapCluster" yaml:"bootstrapCluster"`
|
||||||
AclConfig string `json:"AclConfig" yaml:"AclConfig"`
|
AclConfig string `json:"AclConfig" yaml:"AclConfig"`
|
||||||
|
ForwardCommand bool `json:"forwardCommand" yaml:"forwardCommand"`
|
||||||
RequirePass bool `json:"requirePass" yaml:"requirePass"`
|
RequirePass bool `json:"requirePass" yaml:"requirePass"`
|
||||||
Password string `json:"password" yaml:"password"`
|
Password string `json:"password" yaml:"password"`
|
||||||
}
|
}
|
||||||
@@ -42,10 +43,14 @@ func GetConfig() (Config, error) {
|
|||||||
bindAddr := flag.String("bindAddr", "", "Address to bind the server to.")
|
bindAddr := flag.String("bindAddr", "", "Address to bind the server to.")
|
||||||
raftBindPort := flag.Int("raftPort", 7481, "Port to use for intra-cluster communication. Leave on the client.")
|
raftBindPort := flag.Int("raftPort", 7481, "Port to use for intra-cluster communication. Leave on the client.")
|
||||||
mlBindPort := flag.Int("mlPort", 7946, "Port to use for memberlist communication.")
|
mlBindPort := flag.Int("mlPort", 7946, "Port to use for memberlist communication.")
|
||||||
inMemory := flag.Bool("inMemory", false, "Whether to use memory or persisten storage for raft logs and snapshots.")
|
inMemory := flag.Bool("inMemory", false, "Whether to use memory or persistent storage for raft logs and snapshots.")
|
||||||
dataDir := flag.String("dataDir", "/var/lib/memstore", "Directory to store raft snapshots and logs.")
|
dataDir := flag.String("dataDir", "/var/lib/memstore", "Directory to store raft snapshots and logs.")
|
||||||
bootstrapCluster := flag.Bool("bootstrapCluster", false, "Whether this instance should bootstrap a new cluster.")
|
bootstrapCluster := flag.Bool("bootstrapCluster", false, "Whether this instance should bootstrap a new cluster.")
|
||||||
aclConfig := flag.String("aclConfig", "", "ACL config file path.")
|
aclConfig := flag.String("aclConfig", "", "ACL config file path.")
|
||||||
|
forwardCommand := flag.Bool(
|
||||||
|
"forwardCommand",
|
||||||
|
false,
|
||||||
|
"If the node is a follower, this flag forwards mutation command to the leader when set to true")
|
||||||
requirePass := flag.Bool(
|
requirePass := flag.Bool(
|
||||||
"requirePass",
|
"requirePass",
|
||||||
false,
|
false,
|
||||||
@@ -82,6 +87,7 @@ It is a plain text value by default but you can provide a SHA256 hash by adding
|
|||||||
DataDir: *dataDir,
|
DataDir: *dataDir,
|
||||||
BootstrapCluster: *bootstrapCluster,
|
BootstrapCluster: *bootstrapCluster,
|
||||||
AclConfig: *aclConfig,
|
AclConfig: *aclConfig,
|
||||||
|
ForwardCommand: *forwardCommand,
|
||||||
RequirePass: *requirePass,
|
RequirePass: *requirePass,
|
||||||
Password: *password,
|
Password: *password,
|
||||||
}
|
}
|
||||||
|
Reference in New Issue
Block a user