Interface ReplicationQueueStorage
- All Known Implementing Classes:
NoopReplicationQueueStorage
,ZKReplicationQueueStorage
Perform read/write to the replication queue storage.
-
Method Summary
Modifier and TypeMethodDescriptionvoid
addHFileRefs
(String peerId, List<Pair<org.apache.hadoop.fs.Path, org.apache.hadoop.fs.Path>> pairs) Add new hfile references to the queue.void
addPeerToHFileRefs
(String peerId) Add a peer to hfile reference queue if peer does not exist.void
addWAL
(ServerName serverName, String queueId, String fileName) Add a new WAL file to the given queue for a given regionserver.claimQueue
(ServerName sourceServerName, String queueId, ServerName destServerName) Change ownership for the queue identified by queueId and belongs to a dead region server.Load all hfile references in all replication queues.Get list of all peers from hfile reference queue.getAllQueues
(ServerName serverName) Get a list of all queues for the specified region server.Load all wals in all replication queues.long
getLastSequenceId
(String encodedRegionName, String peerId) Read the max sequence id of the specific region for a given peer.Get a list of all region servers that have outstanding replication queues.getReplicableHFiles
(String peerId) Get a list of all hfile references in the given peer.getRsNode
(ServerName serverName) Get full znode name for given region serverlong
getWALPosition
(ServerName serverName, String queueId, String fileName) Get the current position for a specific WAL in a given queue for a given regionserver.getWALsInQueue
(ServerName serverName, String queueId) Get a list of all WALs in the given queue on the given region server.void
removeHFileRefs
(String peerId, List<String> files) Remove hfile references from the queue.void
removeLastSequenceIds
(String peerId) Remove all the max sequence id record for the given peer.void
removeLastSequenceIds
(String peerId, List<String> encodedRegionNames) Remove the max sequence id record for the given peer and regions.void
removePeerFromHFileRefs
(String peerId) Remove a peer from hfile reference queue.void
removeQueue
(ServerName serverName, String queueId) Remove a replication queue for a given regionserver.void
removeReplicatorIfQueueIsEmpty
(ServerName serverName) Remove the record of region server if the queue is empty.void
removeWAL
(ServerName serverName, String queueId, String fileName) Remove an WAL file from the given queue for a given regionserver.void
setLastSequenceIds
(String peerId, Map<String, Long> lastSeqIds) Set the max sequence id of a bunch of regions for a given peer.void
setWALPosition
(ServerName serverName, String queueId, String fileName, long position, Map<String, Long> lastSeqIds) Set the current position for a specific WAL in a given queue for a given regionserver.
-
Method Details
-
removeQueue
Remove a replication queue for a given regionserver.- Parameters:
serverName
- the name of the regionserverqueueId
- a String that identifies the queue.- Throws:
ReplicationException
-
addWAL
Add a new WAL file to the given queue for a given regionserver. If the queue does not exist it is created.- Parameters:
serverName
- the name of the regionserverqueueId
- a String that identifies the queue.fileName
- name of the WAL- Throws:
ReplicationException
-
removeWAL
Remove an WAL file from the given queue for a given regionserver.- Parameters:
serverName
- the name of the regionserverqueueId
- a String that identifies the queue.fileName
- name of the WAL- Throws:
ReplicationException
-
setWALPosition
void setWALPosition(ServerName serverName, String queueId, String fileName, long position, Map<String, Long> lastSeqIds) throws ReplicationExceptionSet the current position for a specific WAL in a given queue for a given regionserver.- Parameters:
serverName
- the name of the regionserverqueueId
- a String that identifies the queuefileName
- name of the WALposition
- the current position in the file. Will ignore if less than or equal to 0.lastSeqIds
- map with {encodedRegionName, sequenceId} pairs for serial replication.- Throws:
ReplicationException
-
getLastSequenceId
Read the max sequence id of the specific region for a given peer. For serial replication, we need the max sequenced id to decide whether we can push the next entries.- Parameters:
encodedRegionName
- the encoded region namepeerId
- peer id- Returns:
- the max sequence id of the specific region for a given peer.
- Throws:
ReplicationException
-
setLastSequenceIds
Set the max sequence id of a bunch of regions for a given peer. Will be called when setting up a serial replication peer.- Parameters:
peerId
- peer idlastSeqIds
- map with {encodedRegionName, sequenceId} pairs for serial replication.- Throws:
ReplicationException
-
removeLastSequenceIds
Remove all the max sequence id record for the given peer.- Parameters:
peerId
- peer id- Throws:
ReplicationException
-
removeLastSequenceIds
void removeLastSequenceIds(String peerId, List<String> encodedRegionNames) throws ReplicationException Remove the max sequence id record for the given peer and regions.- Parameters:
peerId
- peer idencodedRegionNames
- the encoded region names- Throws:
ReplicationException
-
getWALPosition
long getWALPosition(ServerName serverName, String queueId, String fileName) throws ReplicationException Get the current position for a specific WAL in a given queue for a given regionserver.- Parameters:
serverName
- the name of the regionserverqueueId
- a String that identifies the queuefileName
- name of the WAL- Returns:
- the current position in the file
- Throws:
ReplicationException
-
getWALsInQueue
Get a list of all WALs in the given queue on the given region server.- Parameters:
serverName
- the server name of the region server that owns the queuequeueId
- a String that identifies the queue- Returns:
- a list of WALs
- Throws:
ReplicationException
-
getAllQueues
Get a list of all queues for the specified region server.- Parameters:
serverName
- the server name of the region server that owns the set of queues- Returns:
- a list of queueIds
- Throws:
ReplicationException
-
claimQueue
Pair<String,SortedSet<String>> claimQueue(ServerName sourceServerName, String queueId, ServerName destServerName) throws ReplicationException Change ownership for the queue identified by queueId and belongs to a dead region server.- Parameters:
sourceServerName
- the name of the dead region serverdestServerName
- the name of the target region serverqueueId
- the id of the queue- Returns:
- the new PeerId and A SortedSet of WALs in its queue
- Throws:
ReplicationException
-
removeReplicatorIfQueueIsEmpty
Remove the record of region server if the queue is empty.- Throws:
ReplicationException
-
getListOfReplicators
Get a list of all region servers that have outstanding replication queues. These servers could be alive, dead or from a previous run of the cluster.- Returns:
- a list of server names
- Throws:
ReplicationException
-
getAllWALs
Load all wals in all replication queues. This method guarantees to return a snapshot which contains all WALs at the start of this call even there is concurrent queue failover. However, some newly created WALs during the call may not be included.- Throws:
ReplicationException
-
addPeerToHFileRefs
Add a peer to hfile reference queue if peer does not exist.- Parameters:
peerId
- peer cluster id to be added- Throws:
ReplicationException
- if fails to add a peer id to hfile reference queue
-
removePeerFromHFileRefs
Remove a peer from hfile reference queue.- Parameters:
peerId
- peer cluster id to be removed- Throws:
ReplicationException
-
addHFileRefs
void addHFileRefs(String peerId, List<Pair<org.apache.hadoop.fs.Path, org.apache.hadoop.fs.Path>> pairs) throws ReplicationExceptionAdd new hfile references to the queue.- Parameters:
peerId
- peer cluster id to which the hfiles need to be replicatedpairs
- list of pairs of { HFile location in staging dir, HFile path in region dir which will be added in the queue }- Throws:
ReplicationException
- if fails to add a hfile reference
-
removeHFileRefs
Remove hfile references from the queue.- Parameters:
peerId
- peer cluster id from which this hfile references needs to be removedfiles
- list of hfile references to be removed- Throws:
ReplicationException
-
getAllPeersFromHFileRefsQueue
Get list of all peers from hfile reference queue.- Returns:
- a list of peer ids
- Throws:
ReplicationException
-
getReplicableHFiles
Get a list of all hfile references in the given peer.- Parameters:
peerId
- a String that identifies the peer- Returns:
- a list of hfile references
- Throws:
ReplicationException
-
getAllHFileRefs
Load all hfile references in all replication queues. This method guarantees to return a snapshot which contains all hfile references at the start of this call. However, some newly created hfile references during the call may not be included.- Throws:
ReplicationException
-
getRsNode
Get full znode name for given region server- Parameters:
serverName
- the name of the region server- Returns:
- full znode name
-