Class QuorumPeer

  • All Implemented Interfaces:
    Runnable, QuorumStats.Provider

    public class QuorumPeer
    extends ZooKeeperThread
    implements QuorumStats.Provider
    This class manages the quorum protocol. There are three states this server can be in:
    1. Leader election - each server will elect a leader (proposing itself as a leader initially).
    2. Follower - the server will synchronize with the leader and replicate any transactions.
    3. Leader - the server will process requests and forward them to followers. A majority of followers must log the request before it can be accepted.
    This class will setup a datagram socket that will always respond with its view of the current leader. The response will take the form of:
     int xid;
    
     long myid;
    
     long leader_id;
    
     long leader_zxid;
     
    The request for the current leader will consist solely of an xid: int xid;
    • Field Detail

      • CONFIG_KEY_KERBEROS_CANONICALIZE_HOST_NAMES

        public static final String CONFIG_KEY_KERBEROS_CANONICALIZE_HOST_NAMES
        See Also:
        Constant Field Values
      • CONFIG_DEFAULT_KERBEROS_CANONICALIZE_HOST_NAMES

        public static final String CONFIG_DEFAULT_KERBEROS_CANONICALIZE_HOST_NAMES
        See Also:
        Constant Field Values
      • CONFIG_DEFAULT_MULTI_ADDRESS_ENABLED

        public static final String CONFIG_DEFAULT_MULTI_ADDRESS_ENABLED
        See Also:
        Constant Field Values
      • CONFIG_KEY_MULTI_ADDRESS_REACHABILITY_CHECK_TIMEOUT_MS

        public static final String CONFIG_KEY_MULTI_ADDRESS_REACHABILITY_CHECK_TIMEOUT_MS
        See Also:
        Constant Field Values
      • CONFIG_KEY_MULTI_ADDRESS_REACHABILITY_CHECK_ENABLED

        public static final String CONFIG_KEY_MULTI_ADDRESS_REACHABILITY_CHECK_ENABLED
        See Also:
        Constant Field Values
      • start_fle

        public long start_fle
      • end_fle

        public long end_fle
      • tickTime

        protected int tickTime
        The number of milliseconds of each tick
      • localSessionsEnabled

        protected boolean localSessionsEnabled
        Whether learners in this quorum should create new sessions as local. False by default to preserve existing behavior.
      • localSessionsUpgradingEnabled

        protected boolean localSessionsUpgradingEnabled
        Whether learners in this quorum should upgrade local sessions to global. Only matters if local sessions are enabled.
      • minSessionTimeout

        protected int minSessionTimeout
        Minimum number of milliseconds to allow for session timeout. A value of -1 indicates unset, use default.
      • maxSessionTimeout

        protected int maxSessionTimeout
        Maximum number of milliseconds to allow for session timeout. A value of -1 indicates unset, use default.
      • clientPortListenBacklog

        protected int clientPortListenBacklog
        The ZooKeeper server's socket backlog length. The number of connections that will be queued to be read before new connections are dropped. A value of one indicates the default backlog will be used.
      • initLimit

        protected volatile int initLimit
        The number of ticks that the initial synchronization phase can take
      • syncLimit

        protected volatile int syncLimit
        The number of ticks that can pass between sending a request and getting an acknowledgment
      • connectToLearnerMasterLimit

        protected volatile int connectToLearnerMasterLimit
        The number of ticks that can pass before retrying to connect to learner master
      • syncEnabled

        protected boolean syncEnabled
        Enables/Disables sync request processor. This option is enabled by default and is to be used with observers.
      • quorumListenOnAllIPs

        protected boolean quorumListenOnAllIPs
        Whether or not to listen on all IPs for the two quorum ports (broadcast and fast leader election).
      • quorumSaslEnableAuth

        protected boolean quorumSaslEnableAuth
        Enable/Disables quorum authentication using sasl. Defaulting to false.
      • quorumServerSaslAuthRequired

        protected boolean quorumServerSaslAuthRequired
        If this is false, quorum peer server will accept another quorum peer client connection even if the authentication did not succeed. This can be used while upgrading ZooKeeper server. Defaulting to false (required).
      • quorumLearnerSaslAuthRequired

        protected boolean quorumLearnerSaslAuthRequired
        If this is false, quorum peer learner will talk to quorum peer server without authentication. This can be used while upgrading ZooKeeper server. Defaulting to false (required).
      • quorumServicePrincipal

        protected String quorumServicePrincipal
        Kerberos quorum service principal. Defaulting to 'zkquorum/localhost'.
      • quorumLearnerLoginContext

        protected String quorumLearnerLoginContext
        Quorum learner login context name in jaas-conf file to read the kerberos security details. Defaulting to 'QuorumLearner'.
      • quorumServerLoginContext

        protected String quorumServerLoginContext
        Quorum server login context name in jaas-conf file to read the kerberos security details. Defaulting to 'QuorumServer'.
      • quorumCnxnThreadsSize

        protected int quorumCnxnThreadsSize
        The maximum number of threads to allow in the connectionExecutors thread pool which will be used to initiate quorum server connections.
      • leader

        public Leader leader
      • SYNC_ENABLED

        public static final String SYNC_ENABLED
        The syncEnabled can also be set via a system property.
        See Also:
        Constant Field Values
    • Method Detail

      • getObserverMasterPort

        public int getObserverMasterPort()
      • setObserverMasterPort

        public void setObserverMasterPort​(int observerMasterPort)
      • isMultiAddressEnabled

        public boolean isMultiAddressEnabled()
      • setMultiAddressEnabled

        public void setMultiAddressEnabled​(boolean multiAddressEnabled)
      • getMultiAddressReachabilityCheckTimeoutMs

        public int getMultiAddressReachabilityCheckTimeoutMs()
      • setMultiAddressReachabilityCheckTimeoutMs

        public void setMultiAddressReachabilityCheckTimeoutMs​(int multiAddressReachabilityCheckTimeoutMs)
      • isMultiAddressReachabilityCheckEnabled

        public boolean isMultiAddressReachabilityCheckEnabled()
      • setMultiAddressReachabilityCheckEnabled

        public void setMultiAddressReachabilityCheckEnabled​(boolean multiAddressReachabilityCheckEnabled)
      • setConfigFileName

        protected void setConfigFileName​(String s)
      • getQuorumSize

        public int getQuorumSize()
      • setJvmPauseMonitor

        public void setJvmPauseMonitor​(JvmPauseMonitor jvmPauseMonitor)
      • getMyId

        public long getMyId()
        get the id of this quorum peer.
      • isSslQuorum

        public boolean isSslQuorum()
      • shouldUsePortUnification

        public boolean shouldUsePortUnification()
      • getCurrentVote

        public Vote getCurrentVote()
      • setCurrentVote

        public void setCurrentVote​(Vote v)
      • setLeaderAddressAndId

        public void setLeaderAddressAndId​(MultipleAddresses addr,
                                          long newId)
      • getLeaderAddress

        public String getLeaderAddress()
      • getLeaderId

        public long getLeaderId()
      • getDetailedPeerState

        public String getDetailedPeerState()
      • reconfigFlagSet

        public void reconfigFlagSet()
      • reconfigFlagClear

        public void reconfigFlagClear()
      • isReconfigStateChange

        public boolean isReconfigStateChange()
      • recreateSocketAddresses

        public void recreateSocketAddresses​(long id)
        Resolves hostname for a given server ID. This method resolves hostname for a given server ID in both quorumVerifer and lastSeenQuorumVerifier. If the server ID matches the local server ID, it also updates myAddrs.
      • start

        public void start()
        Overrides:
        start in class Thread
      • stopLeaderElection

        public void stopLeaderElection()
      • startLeaderElection

        public void startLeaderElection()
      • countParticipants

        protected static int countParticipants​(Map<Long,​QuorumPeer.QuorumServer> peers)
        Count the number of nodes in the map that could be followers.
        Parameters:
        peers -
        Returns:
        The number of followers in the map
      • getLastLoggedZxid

        public long getLastLoggedZxid()
        returns the highest zxid that this host has seen
        Returns:
        the highest zxid for this host
      • createElectionAlgorithm

        protected Election createElectionAlgorithm​(int electionAlgorithm)
      • makeLEStrategy

        protected Election makeLEStrategy()
      • setLeader

        protected void setLeader​(Leader newLeader)
      • setFollower

        protected void setFollower​(Follower newFollower)
      • setObserver

        protected void setObserver​(Observer newObserver)
      • setSuspended

        public void setSuspended​(boolean suspended)
      • run

        public void run()
        Specified by:
        run in interface Runnable
        Overrides:
        run in class Thread
      • shutdown

        public void shutdown()
      • getVotingView

        public Map<Long,​QuorumPeer.QuorumServer> getVotingView()
        Observers are not contained in this view, only nodes with PeerType=PARTICIPANT.
      • getCurrentAndNextConfigVoters

        public Set<Long> getCurrentAndNextConfigVoters()
      • viewContains

        public boolean viewContains​(Long sid)
        Check if a node is in the current view. With static membership, the result of this check will never change; only when dynamic membership is introduced will this be more useful.
      • setMyid

        public void setMyid​(long myid)
        set the id of this quorum peer.
      • setInitialConfig

        public void setInitialConfig​(String initialConfig)
      • getInitialConfig

        public String getInitialConfig()
      • getTickTime

        public int getTickTime()
        Get the number of milliseconds of each tick
      • setTickTime

        public void setTickTime​(int tickTime)
        Set the number of milliseconds of each tick
      • getMaxClientCnxnsPerHost

        public int getMaxClientCnxnsPerHost()
        Maximum number of connections allowed from particular host (ip)
      • areLocalSessionsEnabled

        public boolean areLocalSessionsEnabled()
        Whether local sessions are enabled
      • enableLocalSessions

        public void enableLocalSessions​(boolean flag)
        Whether to enable local sessions
      • isLocalSessionsUpgradingEnabled

        public boolean isLocalSessionsUpgradingEnabled()
        Whether local sessions are allowed to upgrade to global sessions
      • enableLocalSessionsUpgrading

        public void enableLocalSessionsUpgrading​(boolean flag)
        Whether to allow local sessions to upgrade to global sessions
      • getMinSessionTimeout

        public int getMinSessionTimeout()
        minimum session timeout in milliseconds
      • setMinSessionTimeout

        public void setMinSessionTimeout​(int min)
        minimum session timeout in milliseconds
      • getMaxSessionTimeout

        public int getMaxSessionTimeout()
        maximum session timeout in milliseconds
      • setMaxSessionTimeout

        public void setMaxSessionTimeout​(int max)
        maximum session timeout in milliseconds
      • getClientPortListenBacklog

        public int getClientPortListenBacklog()
        The server socket's listen backlog length
      • setClientPortListenBacklog

        public void setClientPortListenBacklog​(int backlog)
        Sets the server socket's listen backlog length.
      • getInitLimit

        public int getInitLimit()
        Get the number of ticks that the initial synchronization phase can take
      • setInitLimit

        public void setInitLimit​(int initLimit)
        Set the number of ticks that the initial synchronization phase can take
      • getTick

        public int getTick()
        Get the current tick
      • getQuorumVerifier

        public QuorumVerifier getQuorumVerifier()
        Return QuorumVerifier object for the last committed configuration.
      • getLastSeenQuorumVerifier

        public QuorumVerifier getLastSeenQuorumVerifier()
        Return QuorumVerifier object for the last proposed configuration.
      • getNextDynamicConfigFilename

        public String getNextDynamicConfigFilename()
      • setLastSeenQuorumVerifier

        public void setLastSeenQuorumVerifier​(QuorumVerifier qv,
                                              boolean writeToDisk)
      • getElectionAlg

        public Election getElectionAlg()
        Get an instance of LeaderElection
      • getSyncLimit

        public int getSyncLimit()
        Get the synclimit
      • setSyncLimit

        public void setSyncLimit​(int syncLimit)
        Set the synclimit
      • getConnectToLearnerMasterLimit

        public int getConnectToLearnerMasterLimit()
        Get the connectToLearnerMasterLimit
      • setConnectToLearnerMasterLimit

        public void setConnectToLearnerMasterLimit​(int connectToLearnerMasterLimit)
        Set the connectToLearnerMasterLimit
      • getSyncEnabled

        public boolean getSyncEnabled()
        Return syncEnabled.
      • setSyncEnabled

        public void setSyncEnabled​(boolean syncEnabled)
        Set syncEnabled.
        Parameters:
        syncEnabled -
      • getElectionType

        public int getElectionType()
        Gets the election type
      • setElectionType

        public void setElectionType​(int electionType)
        Sets the election type
      • getQuorumListenOnAllIPs

        public boolean getQuorumListenOnAllIPs()
      • setQuorumListenOnAllIPs

        public void setQuorumListenOnAllIPs​(boolean quorumListenOnAllIPs)
      • setSecureCnxnFactory

        public void setSecureCnxnFactory​(ServerCnxnFactory secureCnxnFactory)
      • setSslQuorum

        public void setSslQuorum​(boolean sslQuorum)
      • setUsePortUnification

        public void setUsePortUnification​(boolean shouldUsePortUnification)
      • setZooKeeperServer

        public void setZooKeeperServer​(ZooKeeperServer zks)
      • closeAllConnections

        public void closeAllConnections()
      • getClientPort

        public int getClientPort()
      • getSecureClientPort

        public int getSecureClientPort()
      • setTxnFactory

        public void setTxnFactory​(FileTxnSnapLog factory)
      • setZKDatabase

        public void setZKDatabase​(ZKDatabase database)
        set zk database for this node
        Parameters:
        database -
      • initConfigInZKDatabase

        public void initConfigInZKDatabase()
      • isRunning

        public boolean isRunning()
      • getQuorumCnxManager

        public QuorumCnxManager getQuorumCnxManager()
        get reference to QuorumCnxManager
      • processReconfig

        public boolean processReconfig​(QuorumVerifier qv,
                                       Long suggestedLeaderId,
                                       Long zxid,
                                       boolean restartLE)
      • updateElectionVote

        protected void updateElectionVote​(long newEpoch)
        Updates leader election info to avoid inconsistencies when a new server tries to join the ensemble. Here is the inconsistency scenario we try to solve by updating the peer epoch after following leader: Let's say we have an ensemble with 3 servers z1, z2 and z3. 1. z1, z2 were following z3 with peerEpoch to be 0xb8, the new epoch is 0xb9, aka current accepted epoch on disk. 2. z2 get restarted, which will use 0xb9 as it's peer epoch when loading the current accept epoch from disk. 3. z2 received notification from z1 and z3, which is following z3 with epoch 0xb8, so it started following z3 again with peer epoch 0xb8. 4. before z2 successfully connected to z3, z3 get restarted with new epoch 0xb9. 5. z2 will retry around a few round (default 5s) before giving up, meanwhile it will report z3 as leader. 6. z1 restarted, and looking with peer epoch 0xb9. 7. z1 voted z3, and z3 was elected as leader again with peer epoch 0xb9. 8. z2 successfully connected to z3 before giving up, but with peer epoch 0xb8. 9. z1 get restarted, looking for leader with peer epoch 0xba, but cannot join, because z2 is reporting peer epoch 0xb8, while z3 is reporting 0xb9. By updating the election vote after actually following leader, we can avoid this kind of stuck happened. Btw, the zxid and electionEpoch could be inconsistent because of the same reason, it's better to update these as well after syncing with leader, but that required protocol change which is non trivial. This problem is worked around by skipping comparing the zxid and electionEpoch when counting for votes for out of election servers during looking for leader. See https://issues.apache.org/jira/browse/ZOOKEEPER-1732
      • isReconfigEnabled

        public boolean isReconfigEnabled()
      • getSynced_observers_metric

        @Private
        public Integer getSynced_observers_metric()
      • createFromConfig

        public static QuorumPeer createFromConfig​(QuorumPeerConfig config)
                                           throws IOException
        Create a new QuorumPeer and apply all the values per the already-parsed config.
        Parameters:
        config - The appertained quorum peer config.
        Returns:
        A QuorumPeer instantiated with specified peer config. Note this peer is not fully initialized; caller should finish initialization through additional configurations (connection factory settings, etc).
        Throws:
        IOException