2013-12-06 23:43:07 +00:00
|
|
|
package consul
|
|
|
|
|
|
|
|
import (
|
2014-04-04 23:30:06 +00:00
|
|
|
"crypto/tls"
|
2014-06-11 02:12:26 +00:00
|
|
|
"errors"
|
2013-12-06 23:43:07 +00:00
|
|
|
"fmt"
|
|
|
|
"log"
|
2013-12-07 00:35:13 +00:00
|
|
|
"net"
|
|
|
|
"net/rpc"
|
2013-12-06 23:43:07 +00:00
|
|
|
"os"
|
|
|
|
"path/filepath"
|
2014-06-11 02:12:26 +00:00
|
|
|
"reflect"
|
2014-04-29 00:07:59 +00:00
|
|
|
"runtime"
|
2014-02-24 00:37:33 +00:00
|
|
|
"strconv"
|
2013-12-06 23:43:07 +00:00
|
|
|
"sync"
|
|
|
|
"time"
|
2014-06-16 21:36:12 +00:00
|
|
|
|
|
|
|
"github.com/hashicorp/raft"
|
|
|
|
"github.com/hashicorp/raft-mdb"
|
|
|
|
"github.com/hashicorp/serf/serf"
|
2013-12-06 23:43:07 +00:00
|
|
|
)
|
|
|
|
|
2014-03-09 22:18:36 +00:00
|
|
|
// These are the protocol versions that Consul can _understand_. These are
|
|
|
|
// Consul-level protocol versions, that are used to configure the Serf
|
|
|
|
// protocol versions.
|
|
|
|
const (
|
|
|
|
ProtocolVersionMin uint8 = 1
|
2014-05-27 22:45:30 +00:00
|
|
|
ProtocolVersionMax = 2
|
2014-03-09 22:18:36 +00:00
|
|
|
)
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
const (
|
2014-04-29 00:07:59 +00:00
|
|
|
serfLANSnapshot = "serf/local.snapshot"
|
|
|
|
serfWANSnapshot = "serf/remote.snapshot"
|
|
|
|
raftState = "raft/"
|
|
|
|
snapshotsRetained = 2
|
|
|
|
raftDBSize32bit uint64 = 128 * 1024 * 1024 // Limit Raft log to 128MB
|
|
|
|
raftDBSize64bit uint64 = 8 * 1024 * 1024 * 1024 // Limit Raft log to 8GB
|
2014-05-27 21:33:09 +00:00
|
|
|
|
|
|
|
// serverRPCCache controls how long we keep an idle connection
|
|
|
|
// open to a server
|
|
|
|
serverRPCCache = 2 * time.Minute
|
|
|
|
|
|
|
|
// serverMaxStreams controsl how many idle streams we keep
|
|
|
|
// open to a server
|
|
|
|
serverMaxStreams = 64
|
2013-12-06 23:43:07 +00:00
|
|
|
)
|
|
|
|
|
|
|
|
// Server is Consul server which manages the service discovery,
|
|
|
|
// health checking, DC forwarding, Raft, and multiple Serf pools.
|
|
|
|
type Server struct {
|
|
|
|
config *Config
|
|
|
|
|
2013-12-09 20:09:57 +00:00
|
|
|
// Connection pool to other consul servers
|
|
|
|
connPool *ConnPool
|
|
|
|
|
2014-01-09 23:30:36 +00:00
|
|
|
// Endpoints holds our RPC endpoints
|
|
|
|
endpoints endpoints
|
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// eventChLAN is used to receive events from the
|
|
|
|
// serf cluster in the datacenter
|
|
|
|
eventChLAN chan serf.Event
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// eventChWAN is used to receive events from the
|
|
|
|
// serf cluster that spans datacenters
|
|
|
|
eventChWAN chan serf.Event
|
2013-12-06 23:43:07 +00:00
|
|
|
|
|
|
|
// fsm is the state machine used with Raft to provide
|
|
|
|
// strong consistency.
|
|
|
|
fsm *consulFSM
|
|
|
|
|
2014-04-23 18:39:45 +00:00
|
|
|
// Have we attempted to leave the cluster
|
|
|
|
left bool
|
|
|
|
|
2014-05-27 22:45:19 +00:00
|
|
|
// localConsuls is used to track the known consuls
|
|
|
|
// in the local data center. Used to do leader forwarding.
|
|
|
|
localConsuls map[string]*serverParts
|
|
|
|
localLock sync.RWMutex
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Logger uses the provided LogOutput
|
|
|
|
logger *log.Logger
|
|
|
|
|
|
|
|
// The raft instance is used among Consul nodes within the
|
|
|
|
// DC to protect operations that require strong consistency
|
2013-12-09 22:20:56 +00:00
|
|
|
raft *raft.Raft
|
|
|
|
raftLayer *RaftLayer
|
2013-12-09 23:29:01 +00:00
|
|
|
raftPeers raft.PeerStore
|
2014-04-19 20:31:56 +00:00
|
|
|
raftStore *raftmdb.MDBStore
|
2013-12-09 22:20:56 +00:00
|
|
|
raftTransport *raft.NetworkTransport
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2014-01-10 20:55:55 +00:00
|
|
|
// reconcileCh is used to pass events from the serf handler
|
|
|
|
// into the leader manager, so that the strong state can be
|
|
|
|
// updated
|
|
|
|
reconcileCh chan serf.Member
|
|
|
|
|
2013-12-12 00:24:34 +00:00
|
|
|
// remoteConsuls is used to track the known consuls in
|
|
|
|
// remote data centers. Used to do DC forwarding.
|
2014-05-27 22:07:31 +00:00
|
|
|
remoteConsuls map[string][]*serverParts
|
2013-12-12 00:24:34 +00:00
|
|
|
remoteLock sync.RWMutex
|
|
|
|
|
2013-12-07 00:35:13 +00:00
|
|
|
// rpcListener is used to listen for incoming connections
|
|
|
|
rpcListener net.Listener
|
|
|
|
rpcServer *rpc.Server
|
|
|
|
|
2014-04-04 23:34:23 +00:00
|
|
|
// rpcTLS is the TLS config for incoming TLS requests
|
|
|
|
rpcTLS *tls.Config
|
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// serfLAN is the Serf cluster maintained inside the DC
|
2013-12-06 23:43:07 +00:00
|
|
|
// which contains all the DC nodes
|
2013-12-07 00:05:26 +00:00
|
|
|
serfLAN *serf.Serf
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// serfWAN is the Serf cluster maintained between DC's
|
2013-12-06 23:43:07 +00:00
|
|
|
// which SHOULD only consist of Consul servers
|
2013-12-07 00:05:26 +00:00
|
|
|
serfWAN *serf.Serf
|
2013-12-06 23:43:07 +00:00
|
|
|
|
|
|
|
shutdown bool
|
|
|
|
shutdownCh chan struct{}
|
|
|
|
shutdownLock sync.Mutex
|
|
|
|
}
|
|
|
|
|
2014-01-09 23:30:36 +00:00
|
|
|
// Holds the RPC endpoints
|
|
|
|
type endpoints struct {
|
2014-04-28 21:44:36 +00:00
|
|
|
Catalog *Catalog
|
|
|
|
Health *Health
|
|
|
|
Status *Status
|
|
|
|
KVS *KVS
|
2014-05-16 21:36:14 +00:00
|
|
|
Session *Session
|
2014-04-28 21:44:36 +00:00
|
|
|
Internal *Internal
|
2014-01-09 23:30:36 +00:00
|
|
|
}
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// NewServer is used to construct a new Consul server from the
|
|
|
|
// configuration, potentially returning an error
|
|
|
|
func NewServer(config *Config) (*Server, error) {
|
2014-03-09 22:18:36 +00:00
|
|
|
// Check the protocol version
|
|
|
|
if err := config.CheckVersion(); err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Check for a data directory!
|
|
|
|
if config.DataDir == "" {
|
|
|
|
return nil, fmt.Errorf("Config must provide a DataDir")
|
|
|
|
}
|
|
|
|
|
|
|
|
// Ensure we have a log output
|
|
|
|
if config.LogOutput == nil {
|
|
|
|
config.LogOutput = os.Stderr
|
|
|
|
}
|
|
|
|
|
2014-04-04 23:34:23 +00:00
|
|
|
// Create the tlsConfig for outgoing connections
|
2014-06-22 19:49:51 +00:00
|
|
|
tlsConfig, err := config.OutgoingTLSConfig()
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
2014-04-04 23:30:06 +00:00
|
|
|
}
|
|
|
|
|
2014-04-04 23:34:23 +00:00
|
|
|
// Get the incoming tls config
|
|
|
|
incomingTLS, err := config.IncomingTLSConfig()
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Create a logger
|
|
|
|
logger := log.New(config.LogOutput, "", log.LstdFlags)
|
|
|
|
|
|
|
|
// Create server
|
|
|
|
s := &Server{
|
2013-12-12 00:24:34 +00:00
|
|
|
config: config,
|
2014-05-28 23:32:10 +00:00
|
|
|
connPool: NewPool(config.LogOutput, serverRPCCache, serverMaxStreams, tlsConfig),
|
2013-12-12 00:24:34 +00:00
|
|
|
eventChLAN: make(chan serf.Event, 256),
|
|
|
|
eventChWAN: make(chan serf.Event, 256),
|
2014-05-27 22:45:19 +00:00
|
|
|
localConsuls: make(map[string]*serverParts),
|
2013-12-12 00:24:34 +00:00
|
|
|
logger: logger,
|
2014-01-10 20:55:55 +00:00
|
|
|
reconcileCh: make(chan serf.Member, 32),
|
2014-05-27 22:07:31 +00:00
|
|
|
remoteConsuls: make(map[string][]*serverParts),
|
2013-12-12 00:24:34 +00:00
|
|
|
rpcServer: rpc.NewServer(),
|
2014-04-04 23:34:23 +00:00
|
|
|
rpcTLS: incomingTLS,
|
2013-12-12 00:24:34 +00:00
|
|
|
shutdownCh: make(chan struct{}),
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
|
|
|
|
2014-06-11 17:17:58 +00:00
|
|
|
// Initialize the RPC layer
|
|
|
|
if err := s.setupRPC(tlsConfig); err != nil {
|
|
|
|
s.Shutdown()
|
|
|
|
return nil, fmt.Errorf("Failed to start RPC layer: %v", err)
|
|
|
|
}
|
|
|
|
|
2013-12-09 23:29:01 +00:00
|
|
|
// Initialize the Raft server
|
|
|
|
if err := s.setupRaft(); err != nil {
|
|
|
|
s.Shutdown()
|
|
|
|
return nil, fmt.Errorf("Failed to start Raft: %v", err)
|
|
|
|
}
|
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// Initialize the lan Serf
|
2013-12-07 01:18:09 +00:00
|
|
|
s.serfLAN, err = s.setupSerf(config.SerfLANConfig,
|
2014-05-16 21:07:50 +00:00
|
|
|
s.eventChLAN, serfLANSnapshot, false)
|
2013-12-06 23:43:07 +00:00
|
|
|
if err != nil {
|
|
|
|
s.Shutdown()
|
2013-12-07 00:05:26 +00:00
|
|
|
return nil, fmt.Errorf("Failed to start lan serf: %v", err)
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
2014-07-22 13:36:58 +00:00
|
|
|
go s.lanEventHandler()
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
// Initialize the wan Serf
|
2013-12-07 01:18:09 +00:00
|
|
|
s.serfWAN, err = s.setupSerf(config.SerfWANConfig,
|
2014-05-16 21:07:50 +00:00
|
|
|
s.eventChWAN, serfWANSnapshot, true)
|
2013-12-06 23:43:07 +00:00
|
|
|
if err != nil {
|
|
|
|
s.Shutdown()
|
2013-12-07 00:05:26 +00:00
|
|
|
return nil, fmt.Errorf("Failed to start wan serf: %v", err)
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
2014-07-22 13:36:58 +00:00
|
|
|
go s.wanEventHandler()
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2014-06-11 17:17:58 +00:00
|
|
|
// Start listening for RPC requests
|
|
|
|
go s.listen()
|
2013-12-06 23:43:07 +00:00
|
|
|
return s, nil
|
|
|
|
}
|
|
|
|
|
|
|
|
// setupSerf is used to setup and initialize a Serf
|
2014-05-16 21:07:50 +00:00
|
|
|
func (s *Server) setupSerf(conf *serf.Config, ch chan serf.Event, path string, wan bool) (*serf.Serf, error) {
|
2013-12-10 00:05:15 +00:00
|
|
|
addr := s.rpcListener.Addr().(*net.TCPAddr)
|
2014-01-30 21:13:29 +00:00
|
|
|
conf.Init()
|
2014-05-16 21:07:50 +00:00
|
|
|
if wan {
|
|
|
|
conf.NodeName = fmt.Sprintf("%s.%s", s.config.NodeName, s.config.Datacenter)
|
|
|
|
} else {
|
|
|
|
conf.NodeName = s.config.NodeName
|
|
|
|
}
|
2014-01-30 21:13:29 +00:00
|
|
|
conf.Tags["role"] = "consul"
|
|
|
|
conf.Tags["dc"] = s.config.Datacenter
|
2014-03-09 22:46:03 +00:00
|
|
|
conf.Tags["vsn"] = fmt.Sprintf("%d", s.config.ProtocolVersion)
|
|
|
|
conf.Tags["vsn_min"] = fmt.Sprintf("%d", ProtocolVersionMin)
|
|
|
|
conf.Tags["vsn_max"] = fmt.Sprintf("%d", ProtocolVersionMax)
|
2014-06-06 22:36:40 +00:00
|
|
|
conf.Tags["build"] = s.config.Build
|
2014-01-30 21:13:29 +00:00
|
|
|
conf.Tags["port"] = fmt.Sprintf("%d", addr.Port)
|
2014-01-20 23:39:07 +00:00
|
|
|
if s.config.Bootstrap {
|
2014-01-30 21:13:29 +00:00
|
|
|
conf.Tags["bootstrap"] = "1"
|
2014-01-20 23:39:07 +00:00
|
|
|
}
|
2014-06-20 00:08:48 +00:00
|
|
|
if s.config.BootstrapExpect != 0 {
|
|
|
|
conf.Tags["expect"] = fmt.Sprintf("%d", s.config.BootstrapExpect)
|
2014-06-18 22:47:05 +00:00
|
|
|
}
|
2013-12-06 23:43:07 +00:00
|
|
|
conf.MemberlistConfig.LogOutput = s.config.LogOutput
|
|
|
|
conf.LogOutput = s.config.LogOutput
|
|
|
|
conf.EventCh = ch
|
|
|
|
conf.SnapshotPath = filepath.Join(s.config.DataDir, path)
|
2014-03-09 22:18:36 +00:00
|
|
|
conf.ProtocolVersion = protocolVersionMap[s.config.ProtocolVersion]
|
2014-05-21 19:32:24 +00:00
|
|
|
conf.RejoinAfterLeave = s.config.RejoinAfterLeave
|
2014-05-16 21:11:53 +00:00
|
|
|
|
|
|
|
// Until Consul supports this fully, we disable automatic resolution.
|
|
|
|
// When enabled, the Serf gossip may just turn off if we are the minority
|
|
|
|
// node which is rather unexpected.
|
|
|
|
conf.EnableNameConflictResolution = false
|
2013-12-19 22:18:55 +00:00
|
|
|
if err := ensurePath(conf.SnapshotPath, false); err != nil {
|
2013-12-06 23:43:07 +00:00
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
return serf.Create(conf)
|
|
|
|
}
|
|
|
|
|
|
|
|
// setupRaft is used to setup and initialize Raft
|
|
|
|
func (s *Server) setupRaft() error {
|
2014-06-18 16:03:30 +00:00
|
|
|
// If we are in bootstrap mode, enable a single node cluster
|
|
|
|
if s.config.Bootstrap {
|
2013-12-25 00:48:07 +00:00
|
|
|
s.config.RaftConfig.EnableSingleNode = true
|
|
|
|
}
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Create the base path
|
|
|
|
path := filepath.Join(s.config.DataDir, raftState)
|
2013-12-19 22:18:55 +00:00
|
|
|
if err := ensurePath(path, true); err != nil {
|
2013-12-06 23:43:07 +00:00
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2013-12-11 01:00:48 +00:00
|
|
|
// Create the FSM
|
|
|
|
var err error
|
2014-02-03 23:21:56 +00:00
|
|
|
s.fsm, err = NewFSM(s.config.LogOutput)
|
2013-12-11 01:00:48 +00:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2014-04-29 00:07:59 +00:00
|
|
|
// Set the maximum raft size based on 32/64bit. Since we are
|
|
|
|
// doing an mmap underneath, we need to limit our use of virtual
|
|
|
|
// address space on 32bit, but don't have to care on 64bit.
|
|
|
|
dbSize := raftDBSize32bit
|
|
|
|
if runtime.GOARCH == "amd64" {
|
|
|
|
dbSize = raftDBSize64bit
|
|
|
|
}
|
|
|
|
|
2013-12-19 00:23:17 +00:00
|
|
|
// Create the MDB store for logs and stable storage
|
2014-04-29 00:07:59 +00:00
|
|
|
store, err := raftmdb.NewMDBStoreWithSize(path, dbSize)
|
2013-12-06 23:43:07 +00:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2013-12-09 21:13:40 +00:00
|
|
|
s.raftStore = store
|
2013-12-06 23:43:07 +00:00
|
|
|
|
|
|
|
// Create the snapshot store
|
2014-02-22 18:57:22 +00:00
|
|
|
snapshots, err := raft.NewFileSnapshotStore(path, snapshotsRetained, s.config.LogOutput)
|
2013-12-06 23:43:07 +00:00
|
|
|
if err != nil {
|
|
|
|
store.Close()
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
|
|
|
// Create a transport layer
|
2014-02-22 19:13:59 +00:00
|
|
|
trans := raft.NewNetworkTransport(s.raftLayer, 3, 10*time.Second, s.config.LogOutput)
|
2013-12-09 22:20:56 +00:00
|
|
|
s.raftTransport = trans
|
2013-12-06 23:43:07 +00:00
|
|
|
|
|
|
|
// Setup the peer store
|
2013-12-09 23:29:01 +00:00
|
|
|
s.raftPeers = raft.NewJSONPeers(path, trans)
|
2013-12-06 23:43:07 +00:00
|
|
|
|
2014-02-21 00:27:14 +00:00
|
|
|
// Ensure local host is always included if we are in bootstrap mode
|
|
|
|
if s.config.Bootstrap {
|
|
|
|
peers, err := s.raftPeers.Peers()
|
|
|
|
if err != nil {
|
|
|
|
store.Close()
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
if !raft.PeerContained(peers, trans.LocalAddr()) {
|
|
|
|
s.raftPeers.SetPeers(raft.AddUniquePeer(peers, trans.LocalAddr()))
|
|
|
|
}
|
2013-12-23 19:50:58 +00:00
|
|
|
}
|
|
|
|
|
2013-12-23 23:30:45 +00:00
|
|
|
// Make sure we set the LogOutput
|
|
|
|
s.config.RaftConfig.LogOutput = s.config.LogOutput
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Setup the Raft store
|
2013-12-09 23:29:01 +00:00
|
|
|
s.raft, err = raft.NewRaft(s.config.RaftConfig, s.fsm, store, store,
|
|
|
|
snapshots, s.raftPeers, trans)
|
2013-12-06 23:43:07 +00:00
|
|
|
if err != nil {
|
|
|
|
store.Close()
|
|
|
|
trans.Close()
|
|
|
|
return err
|
|
|
|
}
|
2014-01-09 23:49:09 +00:00
|
|
|
|
|
|
|
// Start monitoring leadership
|
|
|
|
go s.monitorLeadership()
|
2013-12-06 23:43:07 +00:00
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2013-12-07 00:35:13 +00:00
|
|
|
// setupRPC is used to setup the RPC listener
|
2014-04-04 23:36:47 +00:00
|
|
|
func (s *Server) setupRPC(tlsConfig *tls.Config) error {
|
2014-01-09 23:30:36 +00:00
|
|
|
// Create endpoints
|
|
|
|
s.endpoints.Status = &Status{s}
|
|
|
|
s.endpoints.Catalog = &Catalog{s}
|
|
|
|
s.endpoints.Health = &Health{s}
|
2014-03-31 21:15:49 +00:00
|
|
|
s.endpoints.KVS = &KVS{s}
|
2014-05-16 21:36:14 +00:00
|
|
|
s.endpoints.Session = &Session{s}
|
2014-04-28 21:44:36 +00:00
|
|
|
s.endpoints.Internal = &Internal{s}
|
2014-01-09 23:30:36 +00:00
|
|
|
|
2013-12-09 22:49:07 +00:00
|
|
|
// Register the handlers
|
2014-01-09 23:30:36 +00:00
|
|
|
s.rpcServer.Register(s.endpoints.Status)
|
|
|
|
s.rpcServer.Register(s.endpoints.Catalog)
|
|
|
|
s.rpcServer.Register(s.endpoints.Health)
|
2014-03-31 21:15:49 +00:00
|
|
|
s.rpcServer.Register(s.endpoints.KVS)
|
2014-05-16 21:36:14 +00:00
|
|
|
s.rpcServer.Register(s.endpoints.Session)
|
2014-04-28 21:44:36 +00:00
|
|
|
s.rpcServer.Register(s.endpoints.Internal)
|
2013-12-09 22:49:07 +00:00
|
|
|
|
2014-01-01 00:45:13 +00:00
|
|
|
list, err := net.ListenTCP("tcp", s.config.RPCAddr)
|
2013-12-07 00:35:13 +00:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
s.rpcListener = list
|
2013-12-31 22:00:25 +00:00
|
|
|
|
|
|
|
var advertise net.Addr
|
|
|
|
if s.config.RPCAdvertise != nil {
|
|
|
|
advertise = s.config.RPCAdvertise
|
|
|
|
} else {
|
|
|
|
advertise = s.rpcListener.Addr()
|
|
|
|
}
|
|
|
|
|
|
|
|
// Verify that we have a usable advertise address
|
|
|
|
addr, ok := advertise.(*net.TCPAddr)
|
|
|
|
if !ok {
|
|
|
|
list.Close()
|
|
|
|
return fmt.Errorf("RPC advertise address is not a TCP Address: %v", addr)
|
|
|
|
}
|
|
|
|
if addr.IP.IsUnspecified() {
|
|
|
|
list.Close()
|
|
|
|
return fmt.Errorf("RPC advertise address is not advertisable: %v", addr)
|
|
|
|
}
|
|
|
|
|
2014-04-04 23:36:47 +00:00
|
|
|
s.raftLayer = NewRaftLayer(advertise, tlsConfig)
|
2013-12-07 00:35:13 +00:00
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
// Shutdown is used to shutdown the server
|
|
|
|
func (s *Server) Shutdown() error {
|
2014-01-10 19:06:11 +00:00
|
|
|
s.logger.Printf("[INFO] consul: shutting down server")
|
2013-12-06 23:43:07 +00:00
|
|
|
s.shutdownLock.Lock()
|
|
|
|
defer s.shutdownLock.Unlock()
|
|
|
|
|
|
|
|
if s.shutdown {
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
s.shutdown = true
|
|
|
|
close(s.shutdownCh)
|
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
if s.serfLAN != nil {
|
|
|
|
s.serfLAN.Shutdown()
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
|
|
|
|
2013-12-07 00:05:26 +00:00
|
|
|
if s.serfWAN != nil {
|
|
|
|
s.serfWAN.Shutdown()
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
if s.raft != nil {
|
2013-12-09 22:20:56 +00:00
|
|
|
s.raftTransport.Close()
|
2013-12-09 21:13:40 +00:00
|
|
|
s.raftLayer.Close()
|
2013-12-19 00:40:32 +00:00
|
|
|
future := s.raft.Shutdown()
|
|
|
|
if err := future.Error(); err != nil {
|
2013-12-24 00:20:51 +00:00
|
|
|
s.logger.Printf("[WARN] consul: Error shutting down raft: %s", err)
|
2013-12-19 00:40:32 +00:00
|
|
|
}
|
2013-12-06 23:43:07 +00:00
|
|
|
s.raftStore.Close()
|
2014-04-23 18:39:45 +00:00
|
|
|
|
|
|
|
// Clear the peer set on a graceful leave to avoid
|
|
|
|
// triggering elections on a rejoin.
|
|
|
|
if s.left {
|
|
|
|
s.raftPeers.SetPeers(nil)
|
|
|
|
}
|
2013-12-06 23:43:07 +00:00
|
|
|
}
|
2013-12-07 00:35:13 +00:00
|
|
|
|
|
|
|
if s.rpcListener != nil {
|
|
|
|
s.rpcListener.Close()
|
|
|
|
}
|
|
|
|
|
2013-12-09 20:09:57 +00:00
|
|
|
// Close the connection pool
|
|
|
|
s.connPool.Shutdown()
|
|
|
|
|
2014-02-05 19:00:10 +00:00
|
|
|
// Close the fsm
|
2014-02-06 00:19:05 +00:00
|
|
|
if s.fsm != nil {
|
|
|
|
s.fsm.Close()
|
|
|
|
}
|
2014-02-05 19:00:10 +00:00
|
|
|
|
2013-12-06 23:43:07 +00:00
|
|
|
return nil
|
|
|
|
}
|
2013-12-07 01:18:09 +00:00
|
|
|
|
2013-12-09 20:10:27 +00:00
|
|
|
// Leave is used to prepare for a graceful shutdown of the server
|
|
|
|
func (s *Server) Leave() error {
|
2014-01-10 19:06:11 +00:00
|
|
|
s.logger.Printf("[INFO] consul: server starting leave")
|
2014-04-23 18:39:45 +00:00
|
|
|
s.left = true
|
2013-12-09 20:10:27 +00:00
|
|
|
|
|
|
|
// Leave the WAN pool
|
|
|
|
if s.serfWAN != nil {
|
|
|
|
if err := s.serfWAN.Leave(); err != nil {
|
2014-01-10 19:06:11 +00:00
|
|
|
s.logger.Printf("[ERR] consul: failed to leave WAN Serf cluster: %v", err)
|
2013-12-09 20:10:27 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// Leave the LAN pool
|
|
|
|
if s.serfLAN != nil {
|
|
|
|
if err := s.serfLAN.Leave(); err != nil {
|
2014-01-10 19:06:11 +00:00
|
|
|
s.logger.Printf("[ERR] consul: failed to leave LAN Serf cluster: %v", err)
|
2013-12-09 20:10:27 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2013-12-07 01:18:09 +00:00
|
|
|
// JoinLAN is used to have Consul join the inner-DC pool
|
|
|
|
// The target address should be another node inside the DC
|
|
|
|
// listening on the Serf LAN address
|
2013-12-30 20:20:17 +00:00
|
|
|
func (s *Server) JoinLAN(addrs []string) (int, error) {
|
2014-02-21 00:27:03 +00:00
|
|
|
return s.serfLAN.Join(addrs, true)
|
2013-12-07 01:18:09 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// JoinWAN is used to have Consul join the cross-WAN Consul ring
|
|
|
|
// The target address should be another node listening on the
|
|
|
|
// Serf WAN address
|
2013-12-30 20:20:17 +00:00
|
|
|
func (s *Server) JoinWAN(addrs []string) (int, error) {
|
2014-02-21 00:27:03 +00:00
|
|
|
return s.serfWAN.Join(addrs, true)
|
2013-12-07 01:18:09 +00:00
|
|
|
}
|
2013-12-10 21:17:54 +00:00
|
|
|
|
2014-05-25 23:59:48 +00:00
|
|
|
// LocalMember is used to return the local node
|
|
|
|
func (c *Server) LocalMember() serf.Member {
|
|
|
|
return c.serfLAN.LocalMember()
|
|
|
|
}
|
|
|
|
|
2013-12-10 21:17:54 +00:00
|
|
|
// LANMembers is used to return the members of the LAN cluster
|
|
|
|
func (s *Server) LANMembers() []serf.Member {
|
|
|
|
return s.serfLAN.Members()
|
|
|
|
}
|
|
|
|
|
|
|
|
// WANMembers is used to return the members of the LAN cluster
|
|
|
|
func (s *Server) WANMembers() []serf.Member {
|
|
|
|
return s.serfWAN.Members()
|
|
|
|
}
|
2013-12-10 22:42:29 +00:00
|
|
|
|
|
|
|
// RemoveFailedNode is used to remove a failed node from the cluster
|
|
|
|
func (s *Server) RemoveFailedNode(node string) error {
|
|
|
|
if err := s.serfLAN.RemoveFailedNode(node); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
if err := s.serfWAN.RemoveFailedNode(node); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
2013-12-11 22:04:44 +00:00
|
|
|
|
|
|
|
// IsLeader checks if this server is the cluster leader
|
|
|
|
func (s *Server) IsLeader() bool {
|
|
|
|
return s.raft.State() == raft.Leader
|
|
|
|
}
|
2013-12-19 23:18:25 +00:00
|
|
|
|
2014-06-11 02:12:26 +00:00
|
|
|
// inmemCodec is used to do an RPC call without going over a network
|
|
|
|
type inmemCodec struct {
|
|
|
|
method string
|
|
|
|
args interface{}
|
|
|
|
reply interface{}
|
|
|
|
err error
|
|
|
|
}
|
|
|
|
|
|
|
|
func (i *inmemCodec) ReadRequestHeader(req *rpc.Request) error {
|
|
|
|
req.ServiceMethod = i.method
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
func (i *inmemCodec) ReadRequestBody(args interface{}) error {
|
|
|
|
sourceValue := reflect.Indirect(reflect.Indirect(reflect.ValueOf(i.args)))
|
|
|
|
dst := reflect.Indirect(reflect.Indirect(reflect.ValueOf(args)))
|
|
|
|
dst.Set(sourceValue)
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
func (i *inmemCodec) WriteResponse(resp *rpc.Response, reply interface{}) error {
|
|
|
|
if resp.Error != "" {
|
|
|
|
i.err = errors.New(resp.Error)
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
sourceValue := reflect.Indirect(reflect.Indirect(reflect.ValueOf(reply)))
|
|
|
|
dst := reflect.Indirect(reflect.Indirect(reflect.ValueOf(i.reply)))
|
|
|
|
dst.Set(sourceValue)
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
func (i *inmemCodec) Close() error {
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2013-12-19 23:18:25 +00:00
|
|
|
// RPC is used to make a local RPC call
|
|
|
|
func (s *Server) RPC(method string, args interface{}, reply interface{}) error {
|
2014-06-11 02:12:26 +00:00
|
|
|
codec := &inmemCodec{
|
|
|
|
method: method,
|
|
|
|
args: args,
|
|
|
|
reply: reply,
|
|
|
|
}
|
|
|
|
if err := s.rpcServer.ServeRequest(codec); err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
return codec.err
|
2013-12-19 23:18:25 +00:00
|
|
|
}
|
2014-02-24 00:37:33 +00:00
|
|
|
|
|
|
|
// Stats is used to return statistics for debugging and insight
|
|
|
|
// for various sub-systems
|
|
|
|
func (s *Server) Stats() map[string]map[string]string {
|
|
|
|
toString := func(v uint64) string {
|
|
|
|
return strconv.FormatUint(v, 10)
|
|
|
|
}
|
|
|
|
stats := map[string]map[string]string{
|
|
|
|
"consul": map[string]string{
|
2014-02-24 02:08:58 +00:00
|
|
|
"server": "true",
|
|
|
|
"leader": fmt.Sprintf("%v", s.IsLeader()),
|
|
|
|
"bootstrap": fmt.Sprintf("%v", s.config.Bootstrap),
|
|
|
|
"known_datacenters": toString(uint64(len(s.remoteConsuls))),
|
2014-02-24 00:37:33 +00:00
|
|
|
},
|
2014-02-24 02:08:58 +00:00
|
|
|
"raft": s.raft.Stats(),
|
2014-03-09 22:46:03 +00:00
|
|
|
"serf_lan": s.serfLAN.Stats(),
|
|
|
|
"serf_wan": s.serfWAN.Stats(),
|
2014-04-29 17:55:42 +00:00
|
|
|
"runtime": runtimeStats(),
|
2014-02-24 00:37:33 +00:00
|
|
|
}
|
|
|
|
return stats
|
|
|
|
}
|