001/*
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.replication;
020
021import java.util.List;
022import java.util.SortedSet;
023
024import org.apache.hadoop.fs.Path;
025import org.apache.yetus.audience.InterfaceAudience;
026import org.apache.hadoop.hbase.util.Pair;
027
028/**
029 * This provides an interface for maintaining a region server's replication queues. These queues
030 * keep track of the WALs and HFile references (if hbase.replication.bulkload.enabled is enabled)
031 * that still need to be replicated to remote clusters.
032 */
033@InterfaceAudience.Private
034public interface ReplicationQueues {
035
036  /**
037   * Initialize the region server replication queue interface.
038   * @param serverName The server name of the region server that owns the replication queues this
039   *          interface manages.
040   */
041  void init(String serverName) throws ReplicationException;
042
043  /**
044   * Remove a replication queue.
045   * @param queueId a String that identifies the queue.
046   */
047  void removeQueue(String queueId);
048
049  /**
050   * Add a new WAL file to the given queue. If the queue does not exist it is created.
051   * @param queueId a String that identifies the queue.
052   * @param filename name of the WAL
053   */
054  void addLog(String queueId, String filename) throws ReplicationException;
055
056  /**
057   * Remove an WAL file from the given queue.
058   * @param queueId a String that identifies the queue.
059   * @param filename name of the WAL
060   */
061  void removeLog(String queueId, String filename);
062
063  /**
064   * Set the current position for a specific WAL in a given queue.
065   * @param queueId a String that identifies the queue
066   * @param filename name of the WAL
067   * @param position the current position in the file
068   */
069  void setLogPosition(String queueId, String filename, long position);
070
071  /**
072   * Get the current position for a specific WAL in a given queue.
073   * @param queueId a String that identifies the queue
074   * @param filename name of the WAL
075   * @return the current position in the file
076   */
077  long getLogPosition(String queueId, String filename) throws ReplicationException;
078
079  /**
080   * Remove all replication queues for this region server.
081   */
082  void removeAllQueues();
083
084  /**
085   * Get a list of all WALs in the given queue.
086   * @param queueId a String that identifies the queue
087   * @return a list of WALs, null if no such queue exists for this server
088   */
089  List<String> getLogsInQueue(String queueId);
090
091  /**
092   * Get a list of all queues for this region server.
093   * @return a list of queueIds, an empty list if this region server is dead and has no outstanding queues
094   */
095  List<String> getAllQueues();
096
097  /**
098   * Get queueIds from a dead region server, whose queues has not been claimed by other region
099   * servers.
100   * @return empty if the queue exists but no children, null if the queue does not exist.
101  */
102  List<String> getUnClaimedQueueIds(String regionserver);
103
104  /**
105   * Take ownership for the queue identified by queueId and belongs to a dead region server.
106   * @param regionserver the id of the dead region server
107   * @param queueId the id of the queue
108   * @return the new PeerId and A SortedSet of WALs in its queue, and null if no unclaimed queue.
109   */
110  Pair<String, SortedSet<String>> claimQueue(String regionserver, String queueId);
111
112  /**
113   * Remove the znode of region server if the queue is empty.
114   * @param regionserver
115   */
116  void removeReplicatorIfQueueIsEmpty(String regionserver);
117
118  /**
119   * Get a list of all region servers that have outstanding replication queues. These servers could
120   * be alive, dead or from a previous run of the cluster.
121   * @return a list of server names
122   * @throws ReplicationException
123   */
124  List<String> getListOfReplicators() throws ReplicationException;
125
126  /**
127   * Checks if the provided znode is the same as this region server's
128   * @param regionserver the id of the region server
129   * @return if this is this rs's znode
130   */
131  boolean isThisOurRegionServer(String regionserver);
132
133  /**
134   * Add a peer to hfile reference queue if peer does not exist.
135   * @param peerId peer cluster id to be added
136   * @throws ReplicationException if fails to add a peer id to hfile reference queue
137   */
138  void addPeerToHFileRefs(String peerId) throws ReplicationException;
139
140  /**
141   * Remove a peer from hfile reference queue.
142   * @param peerId peer cluster id to be removed
143   */
144  void removePeerFromHFileRefs(String peerId);
145
146  /**
147   * Add new hfile references to the queue.
148   * @param peerId peer cluster id to which the hfiles need to be replicated
149   * @param pairs list of pairs of { HFile location in staging dir, HFile path in region dir which
150   *          will be added in the queue }
151   * @throws ReplicationException if fails to add a hfile reference
152   */
153  void addHFileRefs(String peerId, List<Pair<Path, Path>> pairs) throws ReplicationException;
154
155  /**
156   * Remove hfile references from the queue.
157   * @param peerId peer cluster id from which this hfile references needs to be removed
158   * @param files list of hfile references to be removed
159   */
160  void removeHFileRefs(String peerId, List<String> files);
161}