Package org.apache.flink.runtime.util
Class ZooKeeperUtils
- java.lang.Object
-
- org.apache.flink.runtime.util.ZooKeeperUtils
-
public class ZooKeeperUtils extends Object
Class containing helper functions to interact with ZooKeeper.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
ZooKeeperUtils.SecureAclProvider
SecureACLProvider
implementation.static class
ZooKeeperUtils.ZkClientACLMode
ZooKeeper client ACL mode enum.
-
Field Summary
Fields Modifier and Type Field Description static String
HA_STORAGE_COMPLETED_CHECKPOINT
The prefix of the completed checkpoint file.static String
HA_STORAGE_SUBMITTED_JOBGRAPH_PREFIX
The prefix of the submitted job graph file.static String
RESOURCE_MANAGER_NODE
The prefix of the resource manager node.
-
Method Summary
All Methods Static Methods Concrete Methods Modifier and Type Method Description static ZooKeeperCheckpointIDCounter
createCheckpointIDCounter(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aZooKeeperCheckpointIDCounter
instance.static CompletedCheckpointStore
createCompletedCheckpoints(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration, int maxNumberOfCheckpointsToRetain, SharedStateRegistryFactory sharedStateRegistryFactory, Executor ioExecutor, Executor executor, RecoveryClaimMode recoveryClaimMode)
Creates aDefaultCompletedCheckpointStore
instance withZooKeeperStateHandleStore
.static <T extends Serializable>
FileSystemStateStorageHelper<T>createFileSystemStateStorage(Configuration configuration, String prefix)
Creates aFileSystemStateStorageHelper
instance.static JobGraphStore
createJobGraphs(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration)
Creates aDefaultJobGraphStore
instance withZooKeeperStateHandleStore
,ZooKeeperJobGraphStoreWatcher
andZooKeeperJobGraphStoreUtil
.static ZooKeeperLeaderRetrievalDriverFactory
createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.static ZooKeeperLeaderRetrievalDriverFactory
createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.static ZooKeeperLeaderRetrievalDriverFactory
createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.static DefaultLeaderRetrievalService
createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aDefaultLeaderRetrievalService
instance withZooKeeperLeaderRetrievalDriver
.static DefaultLeaderRetrievalService
createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
Creates aDefaultLeaderRetrievalService
instance withZooKeeperLeaderRetrievalDriver
.static org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCache
createTreeCache(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String pathToNode, org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCacheSelector selector)
static org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCache
createTreeCache(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String pathToNode, RunnableWithException nodeChangeCallback)
Creates aTreeCache
that only observes a specific node.static <T extends Serializable>
ZooKeeperStateHandleStore<T>createZooKeeperStateHandleStore(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, RetrievableStateStorageHelper<T> stateStorage)
Creates an instance ofZooKeeperStateHandleStore
.static void
deleteZNode(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework curatorFramework, String path)
static String
generateConnectionInformationPath(String path)
static String
generateLeaderLatchPath(String path)
static String
generateZookeeperPath(String... paths)
Creates a ZooKeeper path of the form "/a/b/...static String
getCheckpointIdCounterPath()
static String
getDispatcherNode()
static String
getJobsPath()
static String
getLeaderLatchPath()
static String
getLeaderPath()
static String
getLeaderPath(String suffix)
static String
getLeaderPathForJob(JobID jobId)
static String
getPathForJob(JobID jobId)
Returns the JobID as a String (with leading slash).static String
getResourceManagerNode()
static String
getRestServerNode()
static String
getZooKeeperEnsemble(Configuration flinkConf)
Returns the configured ZooKeeper quorum (and removes whitespace, because ZooKeeper does not tolerate it).static boolean
isConnectionInfoPath(String path)
static boolean
isZooKeeperRecoveryMode(Configuration flinkConf)
Returns whetherHighAvailabilityMode.ZOOKEEPER
is configured.static LeaderInformation
readLeaderInformation(byte[] data)
static String[]
splitZooKeeperPath(String path)
Splits the given ZooKeeper path into its parts.static CuratorFrameworkWithUnhandledErrorListener
startCuratorFramework(Configuration configuration, FatalErrorHandler fatalErrorHandler)
Starts aCuratorFramework
instance and connects it to the given ZooKeeper quorum.static CuratorFrameworkWithUnhandledErrorListener
startCuratorFramework(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFrameworkFactory.Builder builder, FatalErrorHandler fatalErrorHandler)
Starts aCuratorFramework
instance and connects it to the given ZooKeeper quorum from a builder.static String
trimStartingSlash(String path)
static org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework
useNamespaceAndEnsurePath(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path)
Returns a facade of the client that uses the specified namespace, and ensures that all nodes in the path exist.static void
writeLeaderInformationToZooKeeper(LeaderInformation leaderInformation, org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework curatorFramework, BooleanSupplier hasLeadershipCheck, String connectionInformationPath)
-
-
-
Field Detail
-
HA_STORAGE_SUBMITTED_JOBGRAPH_PREFIX
public static final String HA_STORAGE_SUBMITTED_JOBGRAPH_PREFIX
The prefix of the submitted job graph file.- See Also:
- Constant Field Values
-
HA_STORAGE_COMPLETED_CHECKPOINT
public static final String HA_STORAGE_COMPLETED_CHECKPOINT
The prefix of the completed checkpoint file.- See Also:
- Constant Field Values
-
RESOURCE_MANAGER_NODE
public static final String RESOURCE_MANAGER_NODE
The prefix of the resource manager node.- See Also:
- Constant Field Values
-
-
Method Detail
-
getJobsPath
public static String getJobsPath()
-
getCheckpointIdCounterPath
public static String getCheckpointIdCounterPath()
-
getLeaderPath
public static String getLeaderPath()
-
getDispatcherNode
public static String getDispatcherNode()
-
getResourceManagerNode
public static String getResourceManagerNode()
-
getRestServerNode
public static String getRestServerNode()
-
getLeaderLatchPath
public static String getLeaderLatchPath()
-
generateConnectionInformationPath
public static String generateConnectionInformationPath(String path)
-
isConnectionInfoPath
public static boolean isConnectionInfoPath(String path)
-
startCuratorFramework
public static CuratorFrameworkWithUnhandledErrorListener startCuratorFramework(Configuration configuration, FatalErrorHandler fatalErrorHandler)
Starts aCuratorFramework
instance and connects it to the given ZooKeeper quorum.- Parameters:
configuration
-Configuration
object containing the configuration valuesfatalErrorHandler
-FatalErrorHandler
fatalErrorHandler to handle unexpected errors ofCuratorFramework
- Returns:
CuratorFrameworkWithUnhandledErrorListener
instance
-
startCuratorFramework
@VisibleForTesting public static CuratorFrameworkWithUnhandledErrorListener startCuratorFramework(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFrameworkFactory.Builder builder, FatalErrorHandler fatalErrorHandler)
Starts aCuratorFramework
instance and connects it to the given ZooKeeper quorum from a builder.- Parameters:
builder
-CuratorFrameworkFactory.Builder
A builder for curatorFramework.fatalErrorHandler
-FatalErrorHandler
fatalErrorHandler to handle unexpected errors ofCuratorFramework
- Returns:
CuratorFrameworkWithUnhandledErrorListener
instance
-
isZooKeeperRecoveryMode
public static boolean isZooKeeperRecoveryMode(Configuration flinkConf)
Returns whetherHighAvailabilityMode.ZOOKEEPER
is configured.
-
getZooKeeperEnsemble
public static String getZooKeeperEnsemble(Configuration flinkConf) throws IllegalConfigurationException
Returns the configured ZooKeeper quorum (and removes whitespace, because ZooKeeper does not tolerate it).- Throws:
IllegalConfigurationException
-
createLeaderRetrievalService
public static DefaultLeaderRetrievalService createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aDefaultLeaderRetrievalService
instance withZooKeeperLeaderRetrievalDriver
.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to use- Returns:
DefaultLeaderRetrievalService
instance.
-
createLeaderRetrievalService
public static DefaultLeaderRetrievalService createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
Creates aDefaultLeaderRetrievalService
instance withZooKeeperLeaderRetrievalDriver
.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to usepath
- The path for the leader retrievalconfiguration
- configuration for further config options- Returns:
DefaultLeaderRetrievalService
instance.
-
createLeaderRetrievalDriverFactory
public static ZooKeeperLeaderRetrievalDriverFactory createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to use- Returns:
LeaderRetrievalDriverFactory
instance.
-
createLeaderRetrievalDriverFactory
public static ZooKeeperLeaderRetrievalDriverFactory createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to usepath
- The parent path that shall be used by the client.- Returns:
LeaderRetrievalDriverFactory
instance.
-
createLeaderRetrievalDriverFactory
public static ZooKeeperLeaderRetrievalDriverFactory createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
Creates aLeaderRetrievalDriverFactory
implemented by ZooKeeper.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to usepath
- The path for the leader zNodeconfiguration
- configuration for further config options- Returns:
LeaderRetrievalDriverFactory
instance.
-
writeLeaderInformationToZooKeeper
public static void writeLeaderInformationToZooKeeper(LeaderInformation leaderInformation, org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework curatorFramework, BooleanSupplier hasLeadershipCheck, String connectionInformationPath) throws Exception
- Throws:
Exception
-
readLeaderInformation
public static LeaderInformation readLeaderInformation(byte[] data) throws IOException, ClassNotFoundException
- Throws:
IOException
ClassNotFoundException
-
createJobGraphs
public static JobGraphStore createJobGraphs(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration) throws Exception
Creates aDefaultJobGraphStore
instance withZooKeeperStateHandleStore
,ZooKeeperJobGraphStoreWatcher
andZooKeeperJobGraphStoreUtil
.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to useconfiguration
-Configuration
object- Returns:
DefaultJobGraphStore
instance- Throws:
Exception
- if the submitted job graph store cannot be created
-
createCompletedCheckpoints
public static CompletedCheckpointStore createCompletedCheckpoints(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration, int maxNumberOfCheckpointsToRetain, SharedStateRegistryFactory sharedStateRegistryFactory, Executor ioExecutor, Executor executor, RecoveryClaimMode recoveryClaimMode) throws Exception
Creates aDefaultCompletedCheckpointStore
instance withZooKeeperStateHandleStore
.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to useconfiguration
-Configuration
objectmaxNumberOfCheckpointsToRetain
- The maximum number of checkpoints to retainexecutor
- to run ZooKeeper callbacksrecoveryClaimMode
- the mode in which the job is being restored- Returns:
DefaultCompletedCheckpointStore
instance- Throws:
Exception
- if the completed checkpoint store cannot be created
-
getPathForJob
public static String getPathForJob(JobID jobId)
Returns the JobID as a String (with leading slash).
-
createZooKeeperStateHandleStore
public static <T extends Serializable> ZooKeeperStateHandleStore<T> createZooKeeperStateHandleStore(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, RetrievableStateStorageHelper<T> stateStorage) throws Exception
Creates an instance ofZooKeeperStateHandleStore
.- Type Parameters:
T
- Type of state- Parameters:
client
- ZK clientpath
- Path to use for the client namespacestateStorage
- RetrievableStateStorageHelper that persist the actual state and whose returned state handle is then written to ZooKeeper- Returns:
ZooKeeperStateHandleStore
instance- Throws:
Exception
- ZK errors
-
createCheckpointIDCounter
public static ZooKeeperCheckpointIDCounter createCheckpointIDCounter(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates aZooKeeperCheckpointIDCounter
instance.- Parameters:
client
- TheCuratorFramework
ZooKeeper client to use- Returns:
ZooKeeperCheckpointIDCounter
instance
-
createFileSystemStateStorage
public static <T extends Serializable> FileSystemStateStorageHelper<T> createFileSystemStateStorage(Configuration configuration, String prefix) throws IOException
Creates aFileSystemStateStorageHelper
instance.- Type Parameters:
T
- Type of the state objects- Parameters:
configuration
-Configuration
objectprefix
- Prefix for the created files- Returns:
FileSystemStateStorageHelper
instance- Throws:
IOException
- if file system state storage cannot be created
-
generateZookeeperPath
public static String generateZookeeperPath(String... paths)
Creates a ZooKeeper path of the form "/a/b/.../z".
-
splitZooKeeperPath
public static String[] splitZooKeeperPath(String path)
Splits the given ZooKeeper path into its parts.- Parameters:
path
- path to split- Returns:
- splited path
-
useNamespaceAndEnsurePath
public static org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework useNamespaceAndEnsurePath(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path) throws Exception
Returns a facade of the client that uses the specified namespace, and ensures that all nodes in the path exist.- Parameters:
client
- ZK clientpath
- the new namespace- Returns:
- ZK Client that uses the new namespace
- Throws:
Exception
- ZK errors
-
createTreeCache
public static org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCache createTreeCache(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String pathToNode, RunnableWithException nodeChangeCallback)
Creates aTreeCache
that only observes a specific node.- Parameters:
client
- ZK clientpathToNode
- full path of the node to observenodeChangeCallback
- callback to run if the node has changed- Returns:
- tree cache
-
createTreeCache
public static org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCache createTreeCache(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String pathToNode, org.apache.flink.shaded.curator5.org.apache.curator.framework.recipes.cache.TreeCacheSelector selector)
-
-