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.rhkb.cn/news/416398.html

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

相关文章

鸿蒙(API 12 Beta6版)图形【使用Drawing实现图形绘制与显示 (C/C++)】方舟2D图形服务

场景介绍 Native Drawing模块提供了一系列的接口用于基本图形和字体的绘制。 Drawing绘制的内容无法直接在屏幕上显示&#xff0c;需要借用XComponent以及Native Window的能力支持&#xff0c;将绘制的内容通过Native Window送显。 接口说明 Drawing常用接口如下表所示。 …

二分查找:手拿把掐!------Java代码实现

“没有天赋,那就不断重复.” 文章目录 前言文章有误敬请斧正 不胜感恩&#xff01;模板一:(最基本的)**左闭右闭:** [left,right] 模板二:**左闭右开区间模板:**区间:左闭右开[left,right): 模板三:开区间模板:(left,right) 循环不变量:二分查找易错点:做题经验:疑问及解答&…

内衣内裤衣机什么牌子好?五款口碑爆棚王炸机型推荐

如今科技是越来越发展了&#xff0c;迷你洗衣机的功能也是越来越强大了&#xff0c;这样小户型的家庭甚是喜爱&#xff0c;不仅解决了清洗衣物的问题&#xff0c;还能让小型洗衣机在家中起到一定的装饰效果。在清洁衣物的污渍的同时&#xff0c;还能有效除去衣物上的各种细菌。…

upload-labs闯关攻略

pass-1 提前准备好的一个PHP木马&#xff0c;然后将后缀名改为jpg上传 然后在上传的过程中利用抓包&#xff0c;将抓取到的包里面的后缀jpg改为php如图所示&#xff0c;然后放行 接着我们去访问上传的图片信息&#xff0c;如下图所示就为成功 pass-2 提前准备好的一个PHP木马…

http连接处理(最新版)

分析http类及请求接收 基础 epoll epoll_create函数 #include <sys/epoll.h> int epoll_create(int size) 创建一个指示epoll内核事件表的文件描述符&#xff0c;该描述符将用作其他epoll系统调用的第一个参数&#xff0c;size不起作用。 epoll_ctl函数 #include …

紫光同创——PLL IP 的使用(Logos2)

本文档主要针对 Logos2 系列的 PLL 配置&#xff0c;至于 Logos 系列的 PLL&#xff0c;可以参考《PLLIP 的使用(Logos)》的文档。 一、PLL IP 介绍 1、PLL 基本配置模式 Basic Configurations PLL IP 是紫光同创基于 PLL 及时钟网络资源设计的 IP&#xff0c;通过不同的参数配…

牛客周赛 Round 58(ABCDF)

目录 A.会赢吗&#xff1f; B.能做到的吧 C.会赢的&#xff01; D.好好好数 F.随机化游戏时间 A.会赢吗&#xff1f; 思路&#xff1a; 签到题&#xff0c;比大小 void solve() {double a,b;cin>>a>>b;if(a>b) cout<<"NO";else cout<&…

ByteTrack多目标跟踪(一)—理论基础

ByteTrack多目标跟踪 算法概述 github: https://github.com/ifzhang/ByteTrack ByteTrack是一种基于Tracking-by-Detection范式的多目标跟踪算法。 先前的多目标追踪算法一般在完成当前帧的目标检测后只会保留置信度比较大的检测框用于进行目标跟踪&#xff0c;比如图中置信度…

思维导图在线制作怎么制作?5个软件教你快速进行思维导图制作

思维导图在线制作怎么制作&#xff1f;5个软件教你快速进行思维导图制作 思维导图是一种用于组织信息、梳理思路和激发创意的可视化工具。在线制作思维导图可以帮助你随时随地进行创作和分享&#xff0c;以下是五款在线思维导图工具&#xff0c;可以帮助你快速进行思维导图的制…

828华为云征文|基于华为云Flexus云服务器X搭建FTP服务器

❀目录 ❀概述❀特点❀环境准备❀安装❀配置文件修改❀创建目录、修改权限❀控制台安全组开启21端口❀工具验证❀总结 ❀概述 FTP文件传输协议是一种在网络中进行文件传输的广泛使用的标准协议。作为网络通信中的基础工具&#xff0c;FTP允许用户通过客户端软件与服务器进行交…

Java技术栈 —— Spark入门(二)之实时WordCount

Java技术栈 —— Spark入门&#xff08;二&#xff09; 一、kafka1.1 创建topic1.2 准备input与查看output 二、spark2.1 spark下的程序文件2.2 用spark-submit提交作业 参考文章&#xff1a; 参考文章或视频链接[1] 《Kafka Spark Stream实时WordCount》 实验环境&#xff…

【AQS源码】深入理解AQS的工作原理

【AQS源码】深入理解AQS的工作原理-CSDN博客

从零开始掌握容器技术:Docker的奇妙世界

容器技术在当今的云计算和软件开发领域中扮演着越来越重要的角色。如果你是一名计算机专业的学生或从事IT行业的从业者&#xff0c;可能已经听说过Docker这个词。它在软件开发、部署、运维等环节中大放异彩&#xff0c;但对于刚接触这个概念的朋友来说&#xff0c;可能还是有些…

JMeter在Mac下的安装使用

前言 开发过程中需要对系统进行性能测试&#xff0c;可以选用jemter对接口进行压测&#xff0c;jemter优点如下&#xff1a; 开源许可证&#xff1a;Jmeter完全免费&#xff0c;允许开发者使用源代码进行开发 友好的 GUI&#xff1a;Jmeter 非常易于使用&#xff0c;不需要花…

flume 使用 exec 采集容器日志,转储磁盘

flume 使用 exec 采集容器日志&#xff0c;转储磁盘 在该场景下&#xff0c;docker 服务为superset&#xff0c;flume 的sources 选择 exec &#xff0c; sinks选择 file roll 。 任务配置 具体配置文件如下&#xff1a; #simple.conf: A single-node Flume configuration#…

推荐4个一键生成 PPT的AI工具,让你畅享智能办公!

对于职场人士来说&#xff0c;ai PPT 工具已经成为了高效办公的一大得力助手 。它可以让你从繁琐的 PPT 制作中解脱出来&#xff0c;把更多的时间放在其他的工作准备上面。并且它们有极大的设计能力&#xff0c;会让我们的PPT变的设计感十足&#xff0c;如果大家正在为PPT制作烦…

js逆向——RSA实战案例讲解

受害者网站&#xff1a;http://www.15yunmall.com/pc/login/index 检查超时&#xff0c;这个我们不管他 直接分析参数&#xff0c;有2处加密位置&#xff0c;分别为password和csrftoken 只要是能够跟栈的&#xff0c;一律先在send的位置下断 很快就跟栈找到加密数据的位置 R…

《JavaEE进阶》----4.<SpringMVC①简介、基本操作(各种postman请求)>

本篇博客讲解 MVC思想、及Spring MVC&#xff08;是对MVC思想的一种实现&#xff09;。 Spring MVC的基本操作、学习了六个注解 RestController注解 RequestMappering注解 RequestParam注解 RequestBody注解 PathVariable注解 RequestPart注解 MVC View(视图) 指在应⽤程序中…

我用 GPT 学占星

最近对占星赶兴趣&#xff0c;但是看到星盘中好多名词&#xff0c;不懂是什么意思&#xff1f;所以直接问 gpt &#xff0c; 发现回答的真的很棒&#x1f389; &#xff01; 假如我想知道各个状态的具体是根据什么数据来显示的&#xff1f; 分分钟解决了我的问题&#xff1b; 我…

docker Desktop报错 error pulling image configuration 处理

问题描述 在 docker 拉数据 出现以下错误 error pulling image configurarion&#xff1a; 这个问题 主要是 可能应该某些原因不能网络无法连上镜像 原因分析&#xff1a; 1。 2024年 5月以后 国内很多IP都 。。。懂的都懂&#xff0c;很多 VPN 也是。。。 懂的都懂&#x…