public class ZooKeeperUtils extends Object
Modifier and Type | Class and Description |
---|---|
static class |
ZooKeeperUtils.SecureAclProvider
Secure
ACLProvider implementation. |
static class |
ZooKeeperUtils.ZkClientACLMode
ZooKeeper client ACL mode enum.
|
Modifier and Type | Field and Description |
---|---|
static String |
CONNECTION_INFO_NODE |
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.
|
Modifier and Type | Method and Description |
---|---|
static ZooKeeperCheckpointIDCounter |
createCheckpointIDCounter(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates a
ZooKeeperCheckpointIDCounter 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,
RestoreMode restoreMode)
Creates a
DefaultCompletedCheckpointStore instance with ZooKeeperStateHandleStore . |
static <T extends Serializable> |
createFileSystemStateStorage(Configuration configuration,
String prefix)
Creates a
FileSystemStateStorageHelper instance. |
static JobGraphStore |
createJobGraphs(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
Configuration configuration)
Creates a
DefaultJobGraphStore instance with ZooKeeperStateHandleStore ,
ZooKeeperJobGraphStoreWatcher and ZooKeeperJobGraphStoreUtil . |
static ZooKeeperLeaderElectionDriverFactory |
createLeaderElectionDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates a
LeaderElectionDriverFactory implemented by ZooKeeper. |
static ZooKeeperLeaderElectionDriverFactory |
createLeaderElectionDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
String path)
Creates a
LeaderElectionDriverFactory implemented by ZooKeeper. |
static DefaultLeaderElectionService |
createLeaderElectionService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates a
DefaultLeaderElectionService instance with ZooKeeperLeaderElectionDriver . |
static DefaultLeaderElectionService |
createLeaderElectionService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
String path)
Creates a
DefaultLeaderElectionService instance with ZooKeeperLeaderElectionDriver . |
static ZooKeeperLeaderRetrievalDriverFactory |
createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates a
LeaderRetrievalDriverFactory implemented by ZooKeeper. |
static ZooKeeperLeaderRetrievalDriverFactory |
createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
String path,
Configuration configuration)
Creates a
LeaderRetrievalDriverFactory implemented by ZooKeeper. |
static DefaultLeaderRetrievalService |
createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
Creates a
DefaultLeaderRetrievalService instance with ZooKeeperLeaderRetrievalDriver . |
static DefaultLeaderRetrievalService |
createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
String path,
Configuration configuration)
Creates a
DefaultLeaderRetrievalService instance with ZooKeeperLeaderRetrievalDriver . |
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 a
TreeCache that only observes a specific node. |
static <T extends Serializable> |
createZooKeeperStateHandleStore(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client,
String path,
RetrievableStateStorageHelper<T> stateStorage)
Creates an instance of
ZooKeeperStateHandleStore . |
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/.../z".
|
static String |
getCheckpointIdCounterPath() |
static String |
getDispatcherNode() |
static String |
getJobsPath() |
static String |
getLeaderLatchPath() |
static String |
getLeaderPath() |
static String |
getLeaderPathForDispatcher() |
static String |
getLeaderPathForJob(JobID jobId) |
static String |
getLeaderPathForJobManager(JobID jobId) |
static String |
getLeaderPathForResourceManager() |
static String |
getLeaderPathForRestServer() |
static String |
getPathForJob(JobID jobId)
Returns the JobID as a String (with leading slash).
|
static String |
getResourceManagerNode() |
static String |
getRestServerNode() |
static String |
getSingleLeaderElectionPathForJobManager(JobID jobID) |
static String |
getZooKeeperEnsemble(Configuration flinkConf)
Returns the configured ZooKeeper quorum (and removes whitespace, because ZooKeeper does not
tolerate it).
|
static boolean |
isZooKeeperRecoveryMode(Configuration flinkConf)
Returns whether
HighAvailabilityMode.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 a
CuratorFramework 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 a
CuratorFramework 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,
java.util.function.BooleanSupplier hasLeadershipCheck,
String connectionInformationPath) |
public static final String HA_STORAGE_SUBMITTED_JOBGRAPH_PREFIX
public static final String HA_STORAGE_COMPLETED_CHECKPOINT
public static final String CONNECTION_INFO_NODE
public static String getLeaderPathForResourceManager()
public static String getLeaderPathForDispatcher()
public static String getLeaderPathForRestServer()
public static String getSingleLeaderElectionPathForJobManager(JobID jobID)
public static String getJobsPath()
public static String getCheckpointIdCounterPath()
public static String getLeaderPath()
public static String getDispatcherNode()
public static String getResourceManagerNode()
public static String getRestServerNode()
public static String getLeaderLatchPath()
@Nonnull public static String generateConnectionInformationPath(String path)
public static CuratorFrameworkWithUnhandledErrorListener startCuratorFramework(Configuration configuration, FatalErrorHandler fatalErrorHandler)
CuratorFramework
instance and connects it to the given ZooKeeper quorum.configuration
- Configuration
object containing the configuration valuesfatalErrorHandler
- FatalErrorHandler
fatalErrorHandler to handle unexpected
errors of CuratorFramework
CuratorFrameworkWithUnhandledErrorListener
instance@VisibleForTesting public static CuratorFrameworkWithUnhandledErrorListener startCuratorFramework(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFrameworkFactory.Builder builder, FatalErrorHandler fatalErrorHandler)
CuratorFramework
instance and connects it to the given ZooKeeper quorum from
a builder.builder
- CuratorFrameworkFactory.Builder
A builder for curatorFramework.fatalErrorHandler
- FatalErrorHandler
fatalErrorHandler to handle unexpected
errors of CuratorFramework
CuratorFrameworkWithUnhandledErrorListener
instancepublic static boolean isZooKeeperRecoveryMode(Configuration flinkConf)
HighAvailabilityMode.ZOOKEEPER
is configured.public static String getZooKeeperEnsemble(Configuration flinkConf) throws IllegalConfigurationException
IllegalConfigurationException
public static DefaultLeaderRetrievalService createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
DefaultLeaderRetrievalService
instance with ZooKeeperLeaderRetrievalDriver
.client
- The CuratorFramework
ZooKeeper client to useDefaultLeaderRetrievalService
instance.public static DefaultLeaderRetrievalService createLeaderRetrievalService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
DefaultLeaderRetrievalService
instance with ZooKeeperLeaderRetrievalDriver
.client
- The CuratorFramework
ZooKeeper client to usepath
- The path for the leader retrievalconfiguration
- configuration for further config optionsDefaultLeaderRetrievalService
instance.public static ZooKeeperLeaderRetrievalDriverFactory createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
LeaderRetrievalDriverFactory
implemented by ZooKeeper.client
- The CuratorFramework
ZooKeeper client to useLeaderRetrievalDriverFactory
instance.public static ZooKeeperLeaderRetrievalDriverFactory createLeaderRetrievalDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path, Configuration configuration)
LeaderRetrievalDriverFactory
implemented by ZooKeeper.client
- The CuratorFramework
ZooKeeper client to usepath
- The path for the leader zNodeconfiguration
- configuration for further config optionsLeaderRetrievalDriverFactory
instance.public static DefaultLeaderElectionService createLeaderElectionService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
DefaultLeaderElectionService
instance with ZooKeeperLeaderElectionDriver
.client
- The CuratorFramework
ZooKeeper client to useDefaultLeaderElectionService
instance.public static DefaultLeaderElectionService createLeaderElectionService(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path)
DefaultLeaderElectionService
instance with ZooKeeperLeaderElectionDriver
.client
- The CuratorFramework
ZooKeeper client to usepath
- The path for the leader electionDefaultLeaderElectionService
instance.public static ZooKeeperLeaderElectionDriverFactory createLeaderElectionDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
LeaderElectionDriverFactory
implemented by ZooKeeper.client
- The CuratorFramework
ZooKeeper client to useLeaderElectionDriverFactory
instance.public static ZooKeeperLeaderElectionDriverFactory createLeaderElectionDriverFactory(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, String path)
LeaderElectionDriverFactory
implemented by ZooKeeper.client
- The CuratorFramework
ZooKeeper client to usepath
- The path suffix which we want to appendLeaderElectionDriverFactory
instance.public static void writeLeaderInformationToZooKeeper(LeaderInformation leaderInformation, org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework curatorFramework, java.util.function.BooleanSupplier hasLeadershipCheck, String connectionInformationPath) throws Exception
Exception
public static LeaderInformation readLeaderInformation(byte[] data) throws IOException, ClassNotFoundException
IOException
ClassNotFoundException
public static JobGraphStore createJobGraphs(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration) throws Exception
DefaultJobGraphStore
instance with ZooKeeperStateHandleStore
,
ZooKeeperJobGraphStoreWatcher
and ZooKeeperJobGraphStoreUtil
.client
- The CuratorFramework
ZooKeeper client to useconfiguration
- Configuration
objectDefaultJobGraphStore
instanceException
- if the submitted job graph store cannot be createdpublic static CompletedCheckpointStore createCompletedCheckpoints(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client, Configuration configuration, int maxNumberOfCheckpointsToRetain, SharedStateRegistryFactory sharedStateRegistryFactory, Executor ioExecutor, Executor executor, RestoreMode restoreMode) throws Exception
DefaultCompletedCheckpointStore
instance with ZooKeeperStateHandleStore
.client
- The CuratorFramework
ZooKeeper client to useconfiguration
- Configuration
objectmaxNumberOfCheckpointsToRetain
- The maximum number of checkpoints to retainexecutor
- to run ZooKeeper callbacksrestoreMode
- the mode in which the job is being restoredDefaultCompletedCheckpointStore
instanceException
- if the completed checkpoint store cannot be createdpublic static String getPathForJob(JobID jobId)
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
ZooKeeperStateHandleStore
.T
- Type of stateclient
- ZK clientpath
- Path to use for the client namespacestateStorage
- RetrievableStateStorageHelper that persist the actual state and whose
returned state handle is then written to ZooKeeperZooKeeperStateHandleStore
instanceException
- ZK errorspublic static ZooKeeperCheckpointIDCounter createCheckpointIDCounter(org.apache.flink.shaded.curator5.org.apache.curator.framework.CuratorFramework client)
ZooKeeperCheckpointIDCounter
instance.client
- The CuratorFramework
ZooKeeper client to useZooKeeperCheckpointIDCounter
instancepublic static <T extends Serializable> FileSystemStateStorageHelper<T> createFileSystemStateStorage(Configuration configuration, String prefix) throws IOException
FileSystemStateStorageHelper
instance.T
- Type of the state objectsconfiguration
- Configuration
objectprefix
- Prefix for the created filesFileSystemStateStorageHelper
instanceIOException
- if file system state storage cannot be createdpublic static String generateZookeeperPath(String... paths)
public static String[] splitZooKeeperPath(String path)
path
- path to splitpublic 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
client
- ZK clientpath
- the new namespaceException
- ZK errorspublic 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)
TreeCache
that only observes a specific node.client
- ZK clientpathToNode
- full path of the node to observenodeChangeCallback
- callback to run if the node has changedCopyright © 2014–2024 The Apache Software Foundation. All rights reserved.