Apache SeaTunnel Zeta 引擎源码解析(一)Server端的初始化

引入

本系列文章是基于 Apache SeaTunnel 2.3.6版本,围绕Zeta引擎给大家介绍其任务是如何从提交到运行的全流程,希望通过这篇文档,对刚刚上手SeaTunnel的朋友提供一些帮助。

file

我们整体的文章将会分成三篇,从以下方向给大家介绍:

  1. SeaTunnel Server端的初始化
  2. Client端的任务提交流程
  3. Server端的接收到任务的执行流程

由于涉及源码解析,涉及篇幅较大,所以分成系列文章来记录下一个任务的整体流程。

参考

  • [ST-Engine][Design] The Design of LogicalPlan to PhysicalPlan:https://github.com/apache/seatunnel/issues/2269

作者介绍

大家好,我是刘乃杰,一名大数据开发工程师,参与Apache SeaTunnel的开发也有一年多的时间了,不仅给SeaTunnel提交了一些PR,而且添加的一些功能也非常有意思,欢迎大家来找我交流,其中包括支持Avro格式文件,SQL Transform中支持嵌套结构查询,给节点添加Tag达到资源隔离等。

近期推送SeaTunnel在公司内部的落地,需要跟同事,老板介绍SeaTunnel的技术架构,也需要详细的运行流程,帮助同事更好的上手开发,维护。

但是发现目前好像没有这样的一篇文章,能从整体来分析一个任务的执行流程,从而帮助开发者更加容易的定位问题,添加功能。

所以花了一些时间来写了这篇文章,希望抛砖引玉让其他的大佬们也多多写一些源码解析的文章出来。

集群拓扑

首先请大家先从整体了解下SeaTunnel的Zeta引擎架构, SeaTunnel是基于hazelcast来实现的分布式集群通信。

在2.3.6版本之后, 集群中的节点可以被分配为Master或Worker节点,从而将调度与执行分开,避免Master节点的负载过高从而出现问题。

并且2.3.6版本还添加了一个功能,可以对每个节点添加tag属性,当提交任务时可以通过tag来选择任务将要运行的节点, 从而达到资源隔离的目的。

file

集群的服务端分为Master或Worker节点, Master节点负责接收请求、逻辑计划生成、分配任务等(与之前的版本相比,会多了几个Backup节点,但是对于集群稳定性来说是一个挺大的提升)。

而Worker节点则只负责执行任务, 也就是数据的读取和写入。

提交任务时可以创建Hazelcast的客户端连接集群来进行通信,或者使用Restapi来进行通信。

服务端启动

当我们对集群的整体架构有个大致的了解后,再来具体了解下具体的流程。

首先看下Server端的启动过程。

Server端的启动命令为:


sh bin/seatunnel-cluster.sh -d -r <node role type>

查看这个脚本的内容后就会发现, 这个脚本最终的执行命令为:

java -cp seatunnel-starter.jar org.apache.seatunnel.core.starter.seatunnel.SeaTunnelServer <other_java_jvm_config_and_args>

我们查看这个starter.seatunnel.SeaTunnelServer的代码

public class SeaTunnelServer {public static void main(String[] args) throws CommandException {ServerCommandArgs serverCommandArgs =CommandLineUtils.parse(args,new ServerCommandArgs(),EngineType.SEATUNNEL.getStarterShellName(),true);SeaTunnel.run(serverCommandArgs.buildCommand());}
}

这个代码是使用了JCommander来解析用户传递的参数并构建并运行CommandserverCommandArgs.buildCommand返回的类为:

public class ServerExecuteCommand implements Command<ServerCommandArgs> {private final ServerCommandArgs serverCommandArgs;public ServerExecuteCommand(ServerCommandArgs serverCommandArgs) {this.serverCommandArgs = serverCommandArgs;}@Overridepublic void execute() {SeaTunnelConfig seaTunnelConfig = ConfigProvider.locateAndGetSeaTunnelConfig();String clusterRole = this.serverCommandArgs.getClusterRole();if (StringUtils.isNotBlank(clusterRole)) {if (EngineConfig.ClusterRole.MASTER.toString().equalsIgnoreCase(clusterRole)) {seaTunnelConfig.getEngineConfig().setClusterRole(EngineConfig.ClusterRole.MASTER);} else if (EngineConfig.ClusterRole.WORKER.toString().equalsIgnoreCase(clusterRole)) {seaTunnelConfig.getEngineConfig().setClusterRole(EngineConfig.ClusterRole.WORKER);// in hazelcast lite node will not store IMap data.seaTunnelConfig.getHazelcastConfig().setLiteMember(true);} else {throw new SeaTunnelEngineException("Not supported cluster role: " + clusterRole);}} else {seaTunnelConfig.getEngineConfig().setClusterRole(EngineConfig.ClusterRole.MASTER_AND_WORKER);}HazelcastInstanceFactory.newHazelcastInstance(seaTunnelConfig.getHazelcastConfig(),Thread.currentThread().getName(),new SeaTunnelNodeContext(seaTunnelConfig));}
}

在这里会根据配置的角色类型来修改配置信息。

当是Worker节点时,将Hazelcast节点的类型设置为lite member,在Hazelcast中lite member是不进行数据存储的

然后会创建了一个hazelcast实例, 并且传递了SeaTunnelNodeContext实例以及读取并修改的配置信息

public class SeaTunnelNodeContext extends DefaultNodeContext {private final SeaTunnelConfig seaTunnelConfig;public SeaTunnelNodeContext(@NonNull SeaTunnelConfig seaTunnelConfig) {this.seaTunnelConfig = seaTunnelConfig;}@Overridepublic NodeExtension createNodeExtension(@NonNull Node node) {return new org.apache.seatunnel.engine.server.NodeExtension(node, seaTunnelConfig);}@Overridepublic Joiner createJoiner(Node node) {JoinConfig join =getActiveMemberNetworkConfig(seaTunnelConfig.getHazelcastConfig()).getJoin();join.verify();if (node.shouldUseMulticastJoiner(join) && node.multicastService != null) {super.createJoiner(node);} else if (join.getTcpIpConfig().isEnabled()) {log.info("Using LiteNodeDropOutTcpIpJoiner TCP/IP discovery");return new LiteNodeDropOutTcpIpJoiner(node);} else if (node.getProperties().getBoolean(DISCOVERY_SPI_ENABLED)|| isAnyAliasedConfigEnabled(join)|| join.isAutoDetectionEnabled()) {super.createJoiner(node);}return null;}private static boolean isAnyAliasedConfigEnabled(JoinConfig join) {return !AliasedDiscoveryConfigUtils.createDiscoveryStrategyConfigs(join).isEmpty();}private boolean usePublicAddress(JoinConfig join, Node node) {return node.getProperties().getBoolean(DISCOVERY_SPI_PUBLIC_IP_ENABLED)|| allUsePublicAddress(AliasedDiscoveryConfigUtils.aliasedDiscoveryConfigsFrom(join));}
}

SeaTunnelNodeContext中覆盖了createNodeExtension方法, 将使用engine.server.NodeExtension类,

这个类的代码为:

public class NodeExtension extends DefaultNodeExtension {private final NodeExtensionCommon extCommon;public NodeExtension(@NonNull Node node, @NonNull SeaTunnelConfig seaTunnelConfig) {super(node);extCommon = new NodeExtensionCommon(node, new SeaTunnelServer(seaTunnelConfig));}@Overridepublic void beforeStart() {// TODO Get Config from Node heresuper.beforeStart();}@Overridepublic void afterStart() {super.afterStart();extCommon.afterStart();}@Overridepublic void beforeClusterStateChange(ClusterState currState, ClusterState requestedState, boolean isTransient) {super.beforeClusterStateChange(currState, requestedState, isTransient);extCommon.beforeClusterStateChange(requestedState);}@Overridepublic void onClusterStateChange(ClusterState newState, boolean isTransient) {super.onClusterStateChange(newState, isTransient);extCommon.onClusterStateChange(newState);}@Overridepublic Map<String, Object> createExtensionServices() {return extCommon.createExtensionServices();}@Overridepublic TextCommandService createTextCommandService() {return new TextCommandServiceImpl(node) {{register(HTTP_GET, new Log4j2HttpGetCommandProcessor(this));register(HTTP_POST, new Log4j2HttpPostCommandProcessor(this));register(HTTP_GET, new RestHttpGetCommandProcessor(this));register(HTTP_POST, new RestHttpPostCommandProcessor(this));}};}@Overridepublic void printNodeInfo() {extCommon.printNodeInfo(systemLogger);}
}

在这个代码中, 我们可以看到在构造方法中, 初始化了SeaTunnelServer这个类, 而这个类与最开始的类是同名的,

是在不同的包下, 这个类的完整类名为: org.apache.seatunnel.engine.server.SeaTunnelServer

我们看下这个类的代码:

public class SeaTunnelServerimplements ManagedService, MembershipAwareService, LiveOperationsTracker {private static final ILogger LOGGER = Logger.getLogger(SeaTunnelServer.class);public static final String SERVICE_NAME = "st:impl:seaTunnelServer";private NodeEngineImpl nodeEngine;private final LiveOperationRegistry liveOperationRegistry;private volatile SlotService slotService;private TaskExecutionService taskExecutionService;private ClassLoaderService classLoaderService;private CoordinatorService coordinatorService;private ScheduledExecutorService monitorService;@Getter private SeaTunnelHealthMonitor seaTunnelHealthMonitor;private final SeaTunnelConfig seaTunnelConfig;private volatile boolean isRunning = true;public SeaTunnelServer(@NonNull SeaTunnelConfig seaTunnelConfig) {this.liveOperationRegistry = new LiveOperationRegistry();this.seaTunnelConfig = seaTunnelConfig;LOGGER.info("SeaTunnel server start...");}@Overridepublic void init(NodeEngine engine, Properties hzProperties) {...if (EngineConfig.ClusterRole.MASTER_AND_WORKER.ordinal()== seaTunnelConfig.getEngineConfig().getClusterRole().ordinal()) {startWorker();startMaster();} else if (EngineConfig.ClusterRole.WORKER.ordinal()== seaTunnelConfig.getEngineConfig().getClusterRole().ordinal()) {startWorker();} else {startMaster();}...}....
}

这个类是SeaTunnel Server端的核心代码, 在这个类中会根据节点的角色来启动相关的组件。

稍微总结下seatunnel的流程:

SeaTunnel是借助于Hazelcast的基础能力,来实现集群端的组网, 并调用启动核心的代码。

对于这一块有想深入了解的朋友可以去看下Hazelcast的相关内容,这里仅仅列出了调用路径。

按照顺序所加载调用的类为

  1. starter.SeaTunnelServer
  2. ServerExecutreCommand
  3. SeaTunnelNodeContext
  4. NodeExtension
  5. server.SeaTunnelServer

file

接下来再来详细看下Master节点以及Worker节点中所创建的组件

Worker节点

private void startWorker() {taskExecutionService =new TaskExecutionService(classLoaderService, nodeEngine, nodeEngine.getProperties());nodeEngine.getMetricsRegistry().registerDynamicMetricsProvider(taskExecutionService);taskExecutionService.start();getSlotService();
}public SlotService getSlotService() {if (slotService == null) {synchronized (this) {if (slotService == null) {SlotService service =new DefaultSlotService(nodeEngine,taskExecutionService,seaTunnelConfig.getEngineConfig().getSlotServiceConfig());service.init();slotService = service;}}}return slotService;
}

我们可以看到在startWorker方法中, 会初始化两个组件, taskExectutionServiceslotService 这两个组件, 都与任务执行相关。

SlotService

先来看下SlotService的初始化

@Override
public void init() {initStatus = true;slotServiceSequence = UUID.randomUUID().toString();contexts = new ConcurrentHashMap<>();assignedSlots = new ConcurrentHashMap<>();unassignedSlots = new ConcurrentHashMap<>();unassignedResource = new AtomicReference<>(new ResourceProfile());assignedResource = new AtomicReference<>(new ResourceProfile());scheduledExecutorService =Executors.newSingleThreadScheduledExecutor(r ->new Thread(r,String.format("hz.%s.seaTunnel.slotService.thread",nodeEngine.getHazelcastInstance().getName())));if (!config.isDynamicSlot()) {initFixedSlots();}unassignedResource.set(getNodeResource());scheduledExecutorService.scheduleAtFixedRate(() -> {try {LOGGER.fine("start send heartbeat to resource manager, this address: "+ nodeEngine.getClusterService().getThisAddress());sendToMaster(new WorkerHeartbeatOperation(getWorkerProfile())).join();} catch (Exception e) {LOGGER.warning("failed send heartbeat to resource manager, will retry later. this address: "+ nodeEngine.getClusterService().getThisAddress());}},0,DEFAULT_HEARTBEAT_TIMEOUT,TimeUnit.MILLISECONDS);
}

在SeaTunnel中会有一个动态Slot的概念,如果设置为true, 则每个节点就没有Slot的这样一个概念,可以提交任意数量的任务到此节点上,如果设置为固定数量的Slot, 那么该节点仅能接受这些Slot数量的Task运行。

在初始化时,会根据是否为动态Slot来进行数量的初始化。

private void initFixedSlots() {long maxMemory = Runtime.getRuntime().maxMemory();for (int i = 0; i < config.getSlotNum(); i++) {unassignedSlots.put(i,new SlotProfile(nodeEngine.getThisAddress(),i,new ResourceProfile(CPU.of(0), Memory.of(maxMemory / config.getSlotNum())),slotServiceSequence));}
}

同时也可以看到初始化时会启动一个线程,定时向Master节点发送心跳,心跳信息中则包含了当前节点的信息, 包括已经分配的、未分配的Slot数量等属性,Worker节点通过心跳将信息定时更新给Master。

@Override
public synchronized WorkerProfile getWorkerProfile() {WorkerProfile workerProfile = new WorkerProfile(nodeEngine.getThisAddress());workerProfile.setProfile(getNodeResource());workerProfile.setAssignedSlots(assignedSlots.values().toArray(new SlotProfile[0]));workerProfile.setUnassignedSlots(unassignedSlots.values().toArray(new SlotProfile[0]));workerProfile.setUnassignedResource(unassignedResource.get());workerProfile.setAttributes(nodeEngine.getLocalMember().getAttributes());workerProfile.setDynamicSlot(config.isDynamicSlot());return workerProfile;
}private ResourceProfile getNodeResource() {return new ResourceProfile(CPU.of(0), Memory.of(Runtime.getRuntime().maxMemory()));
}

TaskExecutionService

这个组件与任务提交相关, 这里先简单看下,与任务提交的相关代码在后续再深入查看。

在Worker节点初始化时, 会新建一个TaskExecutionService对象,并调用其start方法

private final ExecutorService executorService =newCachedThreadPool(new BlockingTaskThreadFactory());public TaskExecutionService(ClassLoaderService classLoaderService,NodeEngineImpl nodeEngine,HazelcastProperties properties) {// 加载配置信息seaTunnelConfig = ConfigProvider.locateAndGetSeaTunnelConfig();this.hzInstanceName = nodeEngine.getHazelcastInstance().getName();this.nodeEngine = nodeEngine;this.classLoaderService = classLoaderService;this.logger = nodeEngine.getLoggingService().getLogger(TaskExecutionService.class);// 指标相关MetricsRegistry registry = nodeEngine.getMetricsRegistry();MetricDescriptor descriptor =registry.newMetricDescriptor().withTag(MetricTags.SERVICE, this.getClass().getSimpleName());registry.registerStaticMetrics(descriptor, this);// 定时任务更新指标到IMAP中scheduledExecutorService = Executors.newSingleThreadScheduledExecutor();scheduledExecutorService.scheduleAtFixedRate(this::updateMetricsContextInImap,0,seaTunnelConfig.getEngineConfig().getJobMetricsBackupInterval(),TimeUnit.SECONDS);serverConnectorPackageClient =new ServerConnectorPackageClient(nodeEngine, seaTunnelConfig);eventBuffer = new ArrayBlockingQueue<>(2048);// 事件转发服务eventForwardService =Executors.newSingleThreadExecutor(new ThreadFactoryBuilder().setNameFormat("event-forwarder-%d").build());eventForwardService.submit(() -> {List<Event> events = new ArrayList<>();RetryUtils.RetryMaterial retryMaterial =new RetryUtils.RetryMaterial(2, true, e -> true);while (!Thread.currentThread().isInterrupted()) {try {events.clear();Event first = eventBuffer.take();events.add(first);eventBuffer.drainTo(events, 500);JobEventReportOperation operation = new JobEventReportOperation(events);RetryUtils.retryWithException(() ->NodeEngineUtil.sendOperationToMasterNode(nodeEngine, operation).join(),retryMaterial);logger.fine("Event forward success, events " + events.size());} catch (InterruptedException e) {Thread.currentThread().interrupt();logger.info("Event forward thread interrupted");} catch (Throwable t) {logger.warning("Event forward failed, discard events " + events.size(), t);}}});
}public void start() {runBusWorkSupplier.runNewBusWork(false);
}

在这个类中,有一个成员变量,创建了一个线程池。

在构造方法中创建了一个定时任务来更新IMAP里面的任务状态。创建了一个任务来将Event信息发送给Master节点,由Master节点将这些Event发送给外部服务。

file

Master节点

private void startMaster() {coordinatorService =new CoordinatorService(nodeEngine, this, seaTunnelConfig.getEngineConfig());monitorService = Executors.newSingleThreadScheduledExecutor();monitorService.scheduleAtFixedRate(this::printExecutionInfo,0,seaTunnelConfig.getEngineConfig().getPrintExecutionInfoInterval(),TimeUnit.SECONDS);
}

我们可以看到在Master节点中,启动了两个组件:协调器组件和监控组件。

监控组件的任务比较简单, 就是周期性的打印集群信息。

CoordinatorService

public CoordinatorService(@NonNull NodeEngineImpl nodeEngine,@NonNull SeaTunnelServer seaTunnelServer,EngineConfig engineConfig) {this.nodeEngine = nodeEngine;this.logger = nodeEngine.getLogger(getClass());this.executorService =Executors.newCachedThreadPool(new ThreadFactoryBuilder().setNameFormat("seatunnel-coordinator-service-%d").build());this.seaTunnelServer = seaTunnelServer;this.engineConfig = engineConfig;masterActiveListener = Executors.newSingleThreadScheduledExecutor();masterActiveListener.scheduleAtFixedRate(this::checkNewActiveMaster, 0, 100, TimeUnit.MILLISECONDS);
}private void checkNewActiveMaster() {try {if (!isActive && this.seaTunnelServer.isMasterNode()) {logger.info("This node become a new active master node, begin init coordinator service");if (this.executorService.isShutdown()) {this.executorService =Executors.newCachedThreadPool(new ThreadFactoryBuilder().setNameFormat("seatunnel-coordinator-service-%d").build());}initCoordinatorService();isActive = true;} else if (isActive && !this.seaTunnelServer.isMasterNode()) {isActive = false;logger.info("This node become leave active master node, begin clear coordinator service");clearCoordinatorService();}} catch (Exception e) {isActive = false;logger.severe(ExceptionUtils.getMessage(e));throw new SeaTunnelEngineException("check new active master error, stop loop", e);}
}

在初始化时, 会启动一个线程来定时检查当前阶段是否为Master节点, 当节点当前不是Master节点但在集群中成为Master节点时, 会调用initCoordinatorService()来进行状态的初始化, 并将状态修改为True。

当节点自身标记为Master节点,但在集群中已不再是Master节点时,进行状态清理。

private void initCoordinatorService() {// 从hazelcast中获取分布式IMAPrunningJobInfoIMap =nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_RUNNING_JOB_INFO);runningJobStateIMap =nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_RUNNING_JOB_STATE);runningJobStateTimestampsIMap =nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_STATE_TIMESTAMPS);ownedSlotProfilesIMap =nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_OWNED_SLOT_PROFILES);metricsImap = nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_RUNNING_JOB_METRICS);// 初始化JobHistoryServicejobHistoryService =new JobHistoryService(runningJobStateIMap,logger,runningJobMasterMap,nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_FINISHED_JOB_STATE),nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_FINISHED_JOB_METRICS),nodeEngine.getHazelcastInstance().getMap(Constant.IMAP_FINISHED_JOB_VERTEX_INFO),engineConfig.getHistoryJobExpireMinutes());// 初始化EventProcess, 用于发送事件到其他服务eventProcessor =createJobEventProcessor(engineConfig.getEventReportHttpApi(),engineConfig.getEventReportHttpHeaders(),nodeEngine);// If the user has configured the connector package service, create it  on the master node.ConnectorJarStorageConfig connectorJarStorageConfig =engineConfig.getConnectorJarStorageConfig();if (connectorJarStorageConfig.getEnable()) {connectorPackageService = new ConnectorPackageService(seaTunnelServer);}// 集群恢复后, 尝试恢复之前的历史任务restoreAllJobFromMasterNodeSwitchFuture =new PassiveCompletableFuture(CompletableFuture.runAsync(this::restoreAllRunningJobFromMasterNodeSwitch, executorService));
}

Coordinatorservice中,会拉取分布式MAP,这个数据结构是Hazelcast的一个数据结构,可以认为是在集群中数据一致的一个MAP。

在SeaTunnel中, 使用这个结构来存储任务信息、Slot信息等。

在这里还会创建EventProcessor, 这个类是用来将事件通知到其他服务,比如任务失败,可以发送信息到配置的接口中,实现事件推送。

最后,由于节点启动,可能是集群异常重启,或者节点切换,这时需要恢复历史运行的任务,那么就会从刚刚获取到的IMAP中获取到之前正在跑的任务列表,然后尝试进行恢复。

这里的IMAP信息可以开启持久化将信息存储到HDFS等文件系统中, 这样可以在系统完全重启后仍然能够读取到之前的任务状态并进行恢复。

CoordinatorService中运行的组件有:

  • executorService (所有可能被选举为master的节点)
  • jobHistoryService (master节点)
  • eventProcessor (master节点) file

Master节点与备选节点上会:

  1. 定时检查自己是否为Master节点, 如果是则进行相应的状态转化

Master节点上会:

  1. 定时打印集群的状态信息。
  2. 启动转发服务, 将要推送的事件转发到外边服务

在Worker节点上, 启动后会:

  1. 定时将状态信息上报到Master节点
  2. 将任务信息更新到IMAP里面。
  3. 将在Worker产生的要推送给外部服务的事件转发到Master节点上。

至此, 服务端所有服务组件都已启动完成,本文完!

本文由 白鲸开源科技 提供发布支持!

本文来自互联网用户投稿,该文观点仅代表作者本人,不代表本站立场。本站仅提供信息存储空间服务,不拥有所有权,不承担相关法律责任。如若转载,请注明出处:http://www.mzph.cn/diannao/52943.shtml

如若内容造成侵权/违法违规/事实不符,请联系多彩编程网进行投诉反馈email:809451989@qq.com,一经查实,立即删除!

相关文章

指针5.回调函数与qsort

今天来学习回调函数与qsort 目录 1.回调函数实现模拟计算器代码的简化原代码运行结果简化代码运行结果 qsort函数排序整型数据代码运行结果 qsort排序结构数据代码 qsort函数的模拟实现代码运行结果 总结 1.回调函数 回调函数就是⼀个通过函数指针调用的函数。 如果你把函数的…

C++语法基础(一)

第一个C程序 1. <iostream>&#xff08;C&#xff09; <iostream> 是 C 标准库中的头文件&#xff0c;用于处理输入输出操作。它提供了基于流&#xff08;stream&#xff09;的输入输出机制。 特点&#xff1a; 面向对象&#xff1a;C 中的输入输出操作是基于流…

hyperf json-rpc

安装 安装docker hyperf 安装 hyperf-rpc-server-v8 &#xff08;服务端&#xff09; docker run --name hyperf-rpc-server-v8 \ -v /www/docker/hyperf-rpc-server:/data/project \ -w /data/project \ -p 9508:9501 -it \ --privileged -u root \ --entrypoint /bin/sh \…

Upload-LABS通关攻略【1-20关】

Pass-01 第一关是前端JS绕过 上传一个php文件显示只能上传特定后缀名的文件 这里将1.php改为1.jpg直接进行抓包&#xff0c;在数据包中将jpg改为php放行 文件上传成功&#xff0c;邮件图片新建页面打开 可以访问到1.php文件&#xff0c;则一句话密码上传成功 使用蚁剑 进行连接…

Redux的中间件原理分析

Redux的中间件原理分析 redux的中间件对于使用过redux的各位都不会感到陌生&#xff0c;通过应用上我们需要的所有要应用在redux流程上的中间件&#xff0c;我们可以加强dispatch的功能。最近抽了点时间把之前整理分析过的中间件有关的东西放在这里分享分享。本文只对中间件涉…

音视频开发之旅(90)-Vision Transformer论文解读与源码分析

目录 1.背景和问题 2.Vision Transformer(VIT)模型结构 3.Patch Embedding 4.实现效果 5.代码解析 6.资料 一、背景和问题 上一篇我们学习了Transformer的原理&#xff0c;主要介绍了在NLP领域上的应用&#xff0c;那么在CV(图像视频)领域该如何使用&#xff1f; 最直观…

算法复盘——LeetCode hot100:哈希

文章目录 哈希表哈希表的基本概念哈希表的使用1. 插入操作2. 查找操作3. 删除操作 哈希表的优点和缺点1.两数之和复盘 242.有效的字母异位词复盘 49.字母异位词分组复盘 128. 最长连续序列复盘HashSet 哈希表 先来搞清楚什么是哈希表吧~ 概念不清楚方法不清楚怎么做题捏 哈希表…

使用mysql保存密码

登录MySQL 这行命令告诉MySQL客户端程序用户root准备登录&#xff0c;-p表示告诉 MySQL 客户端程序提示输入密码。 mysql -u root -p创建数据库 create database wifi; use wifi;create table password(user_password CHAR(8),primary key(user_password));源码 代码编译 …

QT实战项目之音乐播放器

项目效果演示 myMusicShow 项目概述 在本QT音乐播放器实战项目中&#xff0c;开发环境使用的是QT Creator5.14版本。该项目实现了音乐播放器的基本功能&#xff0c;例如开始播放、停止播放、下一首播放、上一首播放、调节音量、调节倍速、设置音乐播放模式等。同时还具备搜索功…

Centos 下载和 VM 虚拟机安装

1. Centos 下载 阿里云下载地址 centos-7.9.2009-isos-x86_64安装包下载_开源镜像站-阿里云 2. VM 中创建 Centos 虚拟机 2.1 先打开 VM 虚拟机&#xff0c;点击首页的创建新的虚拟机 2.2 选择自定义&#xff0c;然后点击下一步。 2.3 这里默认就好&#xff0c;继续选择下一…

gitlab SSH的使用

一、 安装git bash https://git-scm.com/download/win 下载windows 版本&#xff0c;默认安装即可。 二、使用命令 打开本地git bash,使用如下命令生成ssh公钥和私钥对 ssh-keygen -t rsa -C ‘xxxxxx.com’ 然后一路回车 (-C 参数是你的邮箱地址) 若是想输入密码可以输入…

算法-最长连续序列

leetcode的题目链接 这道题的思路主要是要求在O&#xff08;n)的时间复杂度下&#xff0c;所以你暴力解决肯定不行&#xff0c;暴力至少两层for循环&#xff0c;所以要在O&#xff08;n)的时间复杂度下&#xff0c;你可以使用HashSet来存储数组&#xff0c;对于每个数字&#…

黑马JavaWeb开发笔记07——Ajax、Axios请求、前后端分离开发介绍、Yapi详细配置步骤

文章目录 前言一、Ajax1. 概述2. 作用3. 同步异步4. 原生Ajax请求&#xff08;了解即可&#xff09;5. Axios&#xff08;重点&#xff09;5.1 基本使用5.2 Axios别名&#xff08;简化书写&#xff09; 二、前后端分离开发1. 介绍1.1 前后台混合开发1.2 前后台分离开发方式&…

Docker续6:容器网络

1.bridge-utils 一个用于Linux系统的网络桥接工具集。它提供了一些命令行工具&#xff0c;帮助用户创建、管理和配置网络桥接。网络桥接是一种将多个网络接口连接在一起&#xff0c;以使它们能够作为单个网络段进行通信的技术。 bridge-utils 常用的命令包括&#xff1a; b…

【 OpenHarmony 系统应用源码魔改 】-- Launcher 之「桌面布局定制」

前言 阅读本篇文章之前&#xff0c;有几个需要说明一下&#xff1a; 调试设备&#xff1a;平板&#xff0c;如果你是开发者手机&#xff0c;一样可以加 Log 调试&#xff0c;源码仍然是手机和平板一起分析&#xff1b;文章中的 Log 信息所显示的数值可能跟你的设备不一样&…

单片机编程魔法师-并行多任务程序

程序架构 程序代码 小结 数码分离&#xff0c;本质上就是将数据和代码逻辑进行分离&#xff0c;跟第一章使用数据驱动程序一样的道理。 不过这里不同之处在于。这里使用通过任务线程&#xff0c;但是却有2个任务在运行&#xff0c;两个任务都通过先初始化任务数据参数&#x…

SQLite的安装和使用

一、官网链接下载安装包 点击跳转 步骤&#xff1a;点击安装这个红框的dll以及红框下面的tools &#xff08;如果有navicat可以免上面这个安装步骤&#xff0c;安装上面这个是为了能在命令行敲SQL而已&#xff09; 二、SQLite的特点 嵌入的&#xff08;无服务器的&#x…

hello树先生——AVL树

AVL树 一.什么是AVL树二.AVL树的结构1.AVL树的节点结构2.插入函数3.旋转调整 三.平衡测试 一.什么是AVL树 二叉搜索树虽可以缩短查找的效率&#xff0c;但如果数据有序或接近有序二叉搜索树将退化为单支树&#xff0c;查找元素相当于在顺序表中搜索元素&#xff0c;效率低下。…

python学习——爬虫之session请求处理cookie

import requestssessionrequests.session() url"https://passport.17k.com/ck/user/login" data{"loginName": "19139186287","password":"2001022600hzk"} ressession.post(url,datadata) print(res.text)# session通过会话…

Windows系统中批量管理Windows服务器远程桌面工具——RDCMan

一、背景 在公司没有部署对应的堡垒机系统之前,做运维测试工作的人员,需要管理大量的服务器,每天需要对服务器进行必要的巡检、系统更新发布等内容,特别是有很多Windows服务器的时候,如果我们使用Windows自带的“远程桌面连接”只能一台台连接,比较繁琐。并且不能知道那台…