You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@devlake.apache.org by ab...@apache.org on 2022/09/13 14:46:14 UTC

[incubator-devlake] branch main updated: feat(dora): calculate pr change lead time

This is an automated email from the ASF dual-hosted git repository.

abeizn pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/incubator-devlake.git


The following commit(s) were added to refs/heads/main by this push:
     new cd1df0ac feat(dora): calculate pr change lead time
cd1df0ac is described below

commit cd1df0aca8370bea02cf4646a760cb31660771bd
Author: Yingchu Chen <yi...@merico.dev>
AuthorDate: Thu Sep 8 16:28:11 2022 +0800

    feat(dora): calculate pr change lead time
    
    closes #2878
---
 models/domainlayer/code/pull_request.go            |  40 ++--
 models/domainlayer/devops/cicd_pipeline.go         |  13 ++
 .../20220829_modify_tables_for_dora.go             |  10 +-
 ...dora.go => 20220913_add_origin_value_for_pr.go} |  45 ++---
 models/migrationscripts/register.go                |   1 +
 plugins/dora/dora.go                               |   8 -
 plugins/dora/impl/impl.go                          |   3 +-
 plugins/dora/tasks/change_lead_time_calculator.go  | 202 +++++++++++++++++++++
 plugins/dora/tasks/issue_deploy_connector.go       |   6 +-
 .../e2e/snapshot_tables/cicd_pipeline_repos.csv    |  40 ++--
 plugins/gitlab/e2e/snapshot_tables/cicd_tasks.csv  | 134 +++++++-------
 .../20220906_fix_duration_to_float8.go             |   2 +-
 plugins/gitlab/tasks/commit_convertor.go           |   4 +-
 plugins/gitlab/tasks/job_convertor.go              |  12 +-
 plugins/gitlab/tasks/pipeline_project_convertor.go |   2 +-
 plugins/helper/batch_save.go                       | 105 ++---------
 plugins/helper/batch_save_divider.go               |  81 +--------
 plugins/helper/data_enricher.go                    | 129 -------------
 18 files changed, 382 insertions(+), 455 deletions(-)

diff --git a/models/domainlayer/code/pull_request.go b/models/domainlayer/code/pull_request.go
index 64a1d8c8..126884a8 100644
--- a/models/domainlayer/code/pull_request.go
+++ b/models/domainlayer/code/pull_request.go
@@ -33,24 +33,28 @@ type PullRequest struct {
 	Url         string `gorm:"type:varchar(255)"`
 	AuthorName  string `gorm:"type:varchar(100)"`
 	//User		   domainUser.User `gorm:"foreignKey:AuthorId"`
-	AuthorId       string `gorm:"type:varchar(100)"`
-	ParentPrId     string `gorm:"index;type:varchar(100)"`
-	PullRequestKey int
-	CreatedDate    time.Time
-	MergedDate     *time.Time
-	ClosedDate     *time.Time
-	Type           string `gorm:"type:varchar(100)"`
-	Component      string `gorm:"type:varchar(100)"`
-	MergeCommitSha string `gorm:"type:varchar(40)"`
-	HeadRef        string `gorm:"type:varchar(255)"`
-	BaseRef        string `gorm:"type:varchar(255)"`
-	BaseCommitSha  string `gorm:"type:varchar(40)"`
-	HeadCommitSha  string `gorm:"type:varchar(40)"`
-	CodingTimespan int64
-	ReviewLag      int64
-	ReviewTimespan int64
-	DeployTimespan int64
-	ChangeTimespan int64
+	AuthorId           string `gorm:"type:varchar(100)"`
+	ParentPrId         string `gorm:"index;type:varchar(100)"`
+	PullRequestKey     int
+	CreatedDate        time.Time
+	MergedDate         *time.Time
+	ClosedDate         *time.Time
+	Type               string `gorm:"type:varchar(100)"`
+	Component          string `gorm:"type:varchar(100)"`
+	MergeCommitSha     string `gorm:"type:varchar(40)"`
+	HeadRef            string `gorm:"type:varchar(255)"`
+	BaseRef            string `gorm:"type:varchar(255)"`
+	BaseCommitSha      string `gorm:"type:varchar(40)"`
+	HeadCommitSha      string `gorm:"type:varchar(40)"`
+	CodingTimespan     *int64
+	ReviewLag          *int64
+	ReviewTimespan     *int64
+	DeployTimespan     *int64
+	ChangeTimespan     *int64
+	OrigCodingTimespan int64
+	OrigReviewLag      int64
+	OrigReviewTimespan int64
+	OrigDeployTimespan int64
 }
 
 func (PullRequest) TableName() string {
diff --git a/models/domainlayer/devops/cicd_pipeline.go b/models/domainlayer/devops/cicd_pipeline.go
index fb314c02..f59dd992 100644
--- a/models/domainlayer/devops/cicd_pipeline.go
+++ b/models/domainlayer/devops/cicd_pipeline.go
@@ -57,17 +57,20 @@ const (
 	ABORT       = "ABORT"
 	IN_PROGRESS = "IN_PROGRESS"
 	DONE        = "DONE"
+	MANUAL      = "MANUAL"
 )
 
 type ResultRule struct {
 	Success []string
 	Failed  []string
 	Abort   []string
+	Manual  []string
 	Default string
 }
 type StatusRule struct {
 	InProgress []string
 	Done       []string
+	Manual     []string
 	Default    string
 }
 
@@ -88,6 +91,11 @@ func GetResult(rule *ResultRule, input interface{}) string {
 			return ABORT
 		}
 	}
+	for _, abort := range rule.Manual {
+		if abort == input {
+			return MANUAL
+		}
+	}
 	return rule.Default
 }
 
@@ -103,5 +111,10 @@ func GetStatus(rule *StatusRule, input interface{}) string {
 			return FAILURE
 		}
 	}
+	for _, abort := range rule.Manual {
+		if abort == input {
+			return MANUAL
+		}
+	}
 	return rule.Default
 }
diff --git a/models/migrationscripts/20220829_modify_tables_for_dora.go b/models/migrationscripts/20220829_modify_tables_for_dora.go
index 5c0bcb2c..1340b58c 100644
--- a/models/migrationscripts/20220829_modify_tables_for_dora.go
+++ b/models/migrationscripts/20220829_modify_tables_for_dora.go
@@ -45,11 +45,11 @@ func (*modifyTablesForDora) Name() string {
 }
 
 type PullRequest0829 struct {
-	CodingTimespan uint64
-	ReviewLag      uint64
-	ReviewTimespan uint64
-	DeployTimespan uint64
-	ChangeTimespan uint64
+	CodingTimespan int64
+	ReviewLag      int64
+	ReviewTimespan int64
+	DeployTimespan int64
+	ChangeTimespan int64
 }
 
 func (PullRequest0829) TableName() string {
diff --git a/models/migrationscripts/20220829_modify_tables_for_dora.go b/models/migrationscripts/20220913_add_origin_value_for_pr.go
similarity index 54%
copy from models/migrationscripts/20220829_modify_tables_for_dora.go
copy to models/migrationscripts/20220913_add_origin_value_for_pr.go
index 5c0bcb2c..8391bcf2 100644
--- a/models/migrationscripts/20220829_modify_tables_for_dora.go
+++ b/models/migrationscripts/20220913_add_origin_value_for_pr.go
@@ -22,13 +22,11 @@ import (
 	"gorm.io/gorm"
 )
 
-type modifyTablesForDora struct{}
+type addOriginChangeValueForPr struct{}
 
-func (*modifyTablesForDora) Up(ctx context.Context, db *gorm.DB) error {
+func (*addOriginChangeValueForPr) Up(ctx context.Context, db *gorm.DB) error {
 	err := db.Migrator().AutoMigrate(
-		&CICDPipeline0829{},
-		&PullRequest0829{},
-		&Issue0829{},
+		&PullRequest0913{},
 	)
 	if err != nil {
 		return err
@@ -36,38 +34,21 @@ func (*modifyTablesForDora) Up(ctx context.Context, db *gorm.DB) error {
 	return nil
 }
 
-func (*modifyTablesForDora) Version() uint64 {
-	return 20220829232735
+func (*addOriginChangeValueForPr) Version() uint64 {
+	return 20220913235535
 }
 
-func (*modifyTablesForDora) Name() string {
-	return "modify tables for dora"
+func (*addOriginChangeValueForPr) Name() string {
+	return "add origin change lead time for pr"
 }
 
-type PullRequest0829 struct {
-	CodingTimespan uint64
-	ReviewLag      uint64
-	ReviewTimespan uint64
-	DeployTimespan uint64
-	ChangeTimespan uint64
+type PullRequest0913 struct {
+	OrigCodingTimespan int64
+	OrigReviewLag      int64
+	OrigReviewTimespan int64
+	OrigDeployTimespan int64
 }
 
-func (PullRequest0829) TableName() string {
+func (PullRequest0913) TableName() string {
 	return "pull_requests"
 }
-
-type Issue0829 struct {
-	DeploymentId string `gorm:"type:varchar(255)"`
-}
-
-func (Issue0829) TableName() string {
-	return "issues"
-}
-
-type CICDPipeline0829 struct {
-	Environment string `gorm:"type:varchar(255)"`
-}
-
-func (CICDPipeline0829) TableName() string {
-	return "cicd_pipelines"
-}
diff --git a/models/migrationscripts/register.go b/models/migrationscripts/register.go
index c6a090a4..77c8e71d 100644
--- a/models/migrationscripts/register.go
+++ b/models/migrationscripts/register.go
@@ -44,5 +44,6 @@ func All() []migration.Script {
 		new(encryptPipeline),
 		new(modifyCICDTasks),
 		new(modifyBoardRepos),
+		new(addOriginChangeValueForPr),
 	}
 }
diff --git a/plugins/dora/dora.go b/plugins/dora/dora.go
index 8b8b0fe8..1a63d55d 100644
--- a/plugins/dora/dora.go
+++ b/plugins/dora/dora.go
@@ -33,17 +33,9 @@ func main() {
 	repoId := cmd.Flags().StringP("repoId", "r", "", "repo id")
 	_ = cmd.MarkFlagRequired("repoId")
 
-	// environment := cmd.Flags().StringP("environment", "e", "", "environment")
-	// _ = cmd.MarkFlagRequired("environment")
-
-	// environmentRegex := cmd.Flags().StringP("environmentRegex", "z", "", "environmentRegex")
-	// _ = cmd.MarkFlagRequired("environmentRegex")
-
 	cmd.Run = func(cmd *cobra.Command, args []string) {
 		runner.DirectRun(cmd, args, PluginEntry, map[string]interface{}{
 			"repoId": *repoId,
-			// "environment":      *environment,
-			// "environmentRegex": *environmentRegex,
 		})
 	}
 	runner.RunCmd(cmd)
diff --git a/plugins/dora/impl/impl.go b/plugins/dora/impl/impl.go
index 5432e3b8..8be03a22 100644
--- a/plugins/dora/impl/impl.go
+++ b/plugins/dora/impl/impl.go
@@ -19,7 +19,6 @@ package impl
 
 import (
 	"fmt"
-
 	"github.com/apache/incubator-devlake/errors"
 	"github.com/apache/incubator-devlake/migration"
 	"github.com/apache/incubator-devlake/plugins/core"
@@ -60,7 +59,7 @@ func (plugin Dora) SubTaskMetas() []core.SubTaskMeta {
 	// TODO add your sub task here
 	return []core.SubTaskMeta{
 		tasks.EnrichTaskEnvMeta,
-		//tasks.ConvertChangeLeadTimeMeta,
+		tasks.CalculateChangeLeadTimeMeta,
 		tasks.ConnectIssueDeployMeta,
 	}
 }
diff --git a/plugins/dora/tasks/change_lead_time_calculator.go b/plugins/dora/tasks/change_lead_time_calculator.go
new file mode 100644
index 00000000..a45c87fa
--- /dev/null
+++ b/plugins/dora/tasks/change_lead_time_calculator.go
@@ -0,0 +1,202 @@
+/*
+Licensed to the Apache Software Foundation (ASF) under one or more
+contributor license agreements.  See the NOTICE file distributed with
+this work for additional information regarding copyright ownership.
+The ASF licenses this file to You under the Apache License, Version 2.0
+(the "License"); you may not use this file except in compliance with
+the License.  You may obtain a copy of the License at
+
+    http://www.apache.org/licenses/LICENSE-2.0
+
+Unless required by applicable law or agreed to in writing, software
+distributed under the License is distributed on an "AS IS" BASIS,
+WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+See the License for the specific language governing permissions and
+limitations under the License.
+*/
+
+package tasks
+
+import (
+	"github.com/apache/incubator-devlake/models/domainlayer/code"
+	"github.com/apache/incubator-devlake/models/domainlayer/devops"
+	"github.com/apache/incubator-devlake/plugins/core"
+	"github.com/apache/incubator-devlake/plugins/core/dal"
+	"github.com/apache/incubator-devlake/plugins/helper"
+	"gorm.io/gorm"
+	"reflect"
+	"time"
+)
+
+func CalculateChangeLeadTime(taskCtx core.SubTaskContext) error {
+	data := taskCtx.GetData().(*DoraTaskData)
+	db := taskCtx.GetDal()
+	repoId := data.Options.RepoId
+	clauses := []dal.Clause{
+		dal.From(&code.PullRequest{}),
+		dal.Where("merged_date IS NOT NULL and head_repo_id = ?", repoId),
+	}
+	cursor, err := db.Cursor(clauses...)
+	if err != nil {
+		return err
+	}
+	defer cursor.Close()
+
+	enricher, err := helper.NewDataConverter(helper.DataConverterArgs{
+		RawDataSubTaskArgs: helper.RawDataSubTaskArgs{
+			Ctx:    taskCtx,
+			Params: DoraApiParams{
+				// TODO
+			},
+			Table: "pull_requests",
+		},
+		BatchSize:    100,
+		InputRowType: reflect.TypeOf(code.PullRequest{}),
+		Input:        cursor,
+		Convert: func(inputRow interface{}) ([]interface{}, error) {
+			pr := inputRow.(*code.PullRequest)
+			firstCommitDate, err := getFirstCommitTime(pr.Id, db)
+			if err != nil {
+				return nil, err
+			}
+			if firstCommitDate != nil {
+				codingTime := int64(pr.CreatedDate.Sub(*firstCommitDate).Seconds())
+				if codingTime/60 == 0 && codingTime%60 > 0 {
+					codingTime = 1
+				} else {
+					codingTime = codingTime / 60
+				}
+				pr.OrigCodingTimespan = codingTime
+			}
+			firstReviewTime, err := getFirstReviewTime(pr.Id, pr.AuthorId, db)
+			if err != nil {
+				return nil, err
+			}
+			if firstReviewTime != nil {
+				pr.OrigReviewLag = int64(firstReviewTime.Sub(pr.CreatedDate).Minutes())
+				pr.OrigReviewTimespan = int64(pr.MergedDate.Sub(*firstReviewTime).Minutes())
+			}
+			deployTime, err := getDeployTime(repoId, pr.MergeCommitSha, *pr.MergedDate, db)
+			if err != nil {
+				return nil, err
+			}
+			if deployTime != nil {
+				pr.OrigDeployTimespan = int64(deployTime.Sub(*pr.MergedDate).Minutes())
+			}
+			processNegativeValue(pr)
+			pr.ChangeTimespan = nil
+			result := int64(0)
+			if pr.CodingTimespan != nil {
+				result += *pr.CodingTimespan
+			}
+			if pr.ReviewLag != nil {
+				result += *pr.ReviewLag
+			}
+			if pr.ReviewTimespan != nil {
+				result += *pr.ReviewTimespan
+			}
+			if pr.DeployTimespan != nil {
+				result += *pr.DeployTimespan
+			}
+			if result > 0 {
+				pr.ChangeTimespan = &result
+			}
+			return []interface{}{pr}, nil
+		},
+	})
+	if err != nil {
+		return err
+	}
+
+	return enricher.Execute()
+}
+
+func getFirstCommitTime(prId string, db dal.Dal) (*time.Time, error) {
+	commit := &code.Commit{}
+	commitClauses := []dal.Clause{
+		dal.From(&code.Commit{}),
+		dal.Join("left join pull_request_commits on commits.sha = pull_request_commits.commit_sha"),
+		dal.Where("pull_request_commits.pull_request_id = ?", prId),
+		dal.Orderby("commits.authored_date ASC"),
+	}
+	err := db.First(commit, commitClauses...)
+	if err == gorm.ErrRecordNotFound {
+		return nil, nil
+	}
+	if err != nil {
+		return nil, err
+	}
+	return &commit.AuthoredDate, nil
+}
+
+func getFirstReviewTime(prId string, prCreator string, db dal.Dal) (*time.Time, error) {
+	review := &code.PullRequestComment{}
+	commentClauses := []dal.Clause{
+		dal.From(&code.PullRequestComment{}),
+		dal.Where("pull_request_id = ? and account_id != ?", prId, prCreator),
+		dal.Orderby("created_date ASC"),
+	}
+	err := db.First(review, commentClauses...)
+	if err == gorm.ErrRecordNotFound {
+		return nil, nil
+	}
+	if err != nil {
+		return nil, err
+	}
+	return &review.CreatedDate, nil
+}
+
+func getDeployTime(repoId string, mergeSha string, mergeDate time.Time, db dal.Dal) (*time.Time, error) {
+	cicdTask := &devops.CICDTask{}
+	cicdTaskClauses := []dal.Clause{
+		dal.From(&devops.CICDTask{}),
+		dal.Join("left join cicd_pipelines on cicd_pipelines.id = cicd_tasks.pipeline_id"),
+		dal.Join("left join cicd_pipeline_repos on cicd_pipelines.id = cicd_pipeline_repos.id"),
+		dal.Where(`cicd_pipeline_repos.commit_sha = ? 
+			and cicd_pipeline_repos.repo = ? 
+			and cicd_tasks.type = ? 
+			and cicd_tasks.result = ?
+			and cicd_tasks.started_date > ?`,
+			mergeSha, repoId, "DEPLOY", "SUCCESS", mergeDate),
+		dal.Orderby("cicd_tasks.started_date ASC"),
+	}
+	err := db.First(cicdTask, cicdTaskClauses...)
+	if err == gorm.ErrRecordNotFound {
+		return nil, nil
+	}
+	if err != nil {
+		return nil, err
+	}
+	return cicdTask.FinishedDate, nil
+}
+
+func processNegativeValue(pr *code.PullRequest) {
+	if pr.OrigCodingTimespan > 0 {
+		pr.CodingTimespan = &pr.OrigCodingTimespan
+	} else {
+		pr.CodingTimespan = nil
+	}
+	if pr.OrigReviewLag > 0 {
+		pr.ReviewLag = &pr.OrigReviewLag
+	} else {
+		pr.ReviewLag = nil
+	}
+	if pr.OrigReviewTimespan > 0 {
+		pr.ReviewTimespan = &pr.OrigReviewTimespan
+	} else {
+		pr.ReviewTimespan = nil
+	}
+	if pr.OrigDeployTimespan > 0 {
+		pr.DeployTimespan = &pr.OrigDeployTimespan
+	} else {
+		pr.DeployTimespan = nil
+	}
+}
+
+var CalculateChangeLeadTimeMeta = core.SubTaskMeta{
+	Name:             "calculateChangeLeadTime",
+	EntryPoint:       CalculateChangeLeadTime,
+	EnabledByDefault: true,
+	Description:      "Calculate change lead time",
+	DomainTypes:      []string{core.DOMAIN_TYPE_CICD, core.DOMAIN_TYPE_CODE},
+}
diff --git a/plugins/dora/tasks/issue_deploy_connector.go b/plugins/dora/tasks/issue_deploy_connector.go
index 2681df60..8b1bfa40 100644
--- a/plugins/dora/tasks/issue_deploy_connector.go
+++ b/plugins/dora/tasks/issue_deploy_connector.go
@@ -58,7 +58,7 @@ func ConnectIssueDeploy(taskCtx core.SubTaskContext) error {
 	}
 	defer cursor.Close()
 
-	enricher, err := helper.NewDataEnricher(helper.DataEnricherArgs{
+	enricher, err := helper.NewDataConverter(helper.DataConverterArgs{
 		RawDataSubTaskArgs: helper.RawDataSubTaskArgs{
 			Ctx:    taskCtx,
 			Params: DoraApiParams{
@@ -68,13 +68,13 @@ func ConnectIssueDeploy(taskCtx core.SubTaskContext) error {
 		},
 		InputRowType: reflect.TypeOf(ticket.Issue{}),
 		Input:        cursor,
-		Enrich: func(inputRow interface{}) ([]interface{}, error) {
+		Convert: func(inputRow interface{}) ([]interface{}, error) {
 			issueToBeUpdate := inputRow.(*ticket.Issue)
 			cicdTask := &devops.CICDTask{}
 			cicdTakClauses := []dal.Clause{
 				dal.From(cicdTask),
 				dal.Join(`left join cicd_pipelines 
-			on cicd_pipelines.id = cicd_tasks.pipeline_id`),
+					on cicd_pipelines.id = cicd_tasks.pipeline_id`),
 				dal.Join("left join cicd_pipeline_repos on cicd_pipelines.id = cicd_pipeline_repos.id"),
 				dal.Where(
 					`cicd_pipeline_repos.repo = ? and cicd_tasks.finished_date < ? 
diff --git a/plugins/gitlab/e2e/snapshot_tables/cicd_pipeline_repos.csv b/plugins/gitlab/e2e/snapshot_tables/cicd_pipeline_repos.csv
index 6288aab4..b0130e83 100644
--- a/plugins/gitlab/e2e/snapshot_tables/cicd_pipeline_repos.csv
+++ b/plugins/gitlab/e2e/snapshot_tables/cicd_pipeline_repos.csv
@@ -1,21 +1,21 @@
 id,commit_sha,branch,repo
-gitlab:GitlabPipeline:1:457474837,b1b82852d48b516a18e56c5bab0ebf54b8f4ccfd,renovate/pin-dependencies,gitlab:GitlabProject:1:457474837
-gitlab:GitlabPipeline:1:457474996,739ab912e6e1ed27cecd8a2d00bfd6fa52afd90a,renovate/jest-monorepo,gitlab:GitlabProject:1:457474996
-gitlab:GitlabPipeline:1:457475160,44d127e0ab7dbc4bc259b55929c9d00b62fc3bf4,renovate/lodash-monorepo,gitlab:GitlabProject:1:457475160
-gitlab:GitlabPipeline:1:457475337,10a6464b6bd2cf4b59b8ac37ce1466e013f5a20d,renovate/shx-0.x,gitlab:GitlabProject:1:457475337
-gitlab:GitlabPipeline:1:485811050,c791ea6949d6b4aadf79b15ba666cb690c6527ac,EE-7121,gitlab:GitlabProject:1:485811050
-gitlab:GitlabPipeline:1:485811059,c791ea6949d6b4aadf79b15ba666cb690c6527ac,refs/merge-requests/74/head,gitlab:GitlabProject:1:485811059
-gitlab:GitlabPipeline:1:485813816,ecc7c0b2874c812ed882c9effbbda26e0abc7110,EE-7121,gitlab:GitlabProject:1:485813816
-gitlab:GitlabPipeline:1:485813830,ecc7c0b2874c812ed882c9effbbda26e0abc7110,refs/merge-requests/74/head,gitlab:GitlabProject:1:485813830
-gitlab:GitlabPipeline:1:485814501,6a3346f8434cc65fbe3f7a80a0edec5b4014a733,EE-7121,gitlab:GitlabProject:1:485814501
-gitlab:GitlabPipeline:1:485814516,6a3346f8434cc65fbe3f7a80a0edec5b4014a733,refs/merge-requests/74/head,gitlab:GitlabProject:1:485814516
-gitlab:GitlabPipeline:1:485814871,f731732471961fed061a01fa99631f69e861c4a2,refs/merge-requests/74/head,gitlab:GitlabProject:1:485814871
-gitlab:GitlabPipeline:1:485817670,5b95c5aebce1eae6a1b95ecf6fbc870851455375,refs/merge-requests/74/head,gitlab:GitlabProject:1:485817670
-gitlab:GitlabPipeline:1:485837602,a79e2aaeee9916338a11b665cb4201f24627f3f1,refs/merge-requests/74/head,gitlab:GitlabProject:1:485837602
-gitlab:GitlabPipeline:1:485842553,9e0934cedcd83abab8d34c5bb0f597c7837a55f0,refs/merge-requests/74/head,gitlab:GitlabProject:1:485842553
-gitlab:GitlabPipeline:1:485845850,b3b581940f6d4d3226db31e48eab2b3b73f4c6db,refs/merge-requests/74/head,gitlab:GitlabProject:1:485845850
-gitlab:GitlabPipeline:1:485852752,82df93eeb72c71271dad5a85359f39661de899fe,refs/merge-requests/74/head,gitlab:GitlabProject:1:485852752
-gitlab:GitlabPipeline:1:485865876,db45efd231dbaca1337bd29bc1f65f404754e6bd,refs/merge-requests/74/head,gitlab:GitlabProject:1:485865876
-gitlab:GitlabPipeline:1:485877118,09f81b1b2d083411c0bfecd32d7728479b594503,refs/merge-requests/74/head,gitlab:GitlabProject:1:485877118
-gitlab:GitlabPipeline:1:485905167,cce432655861efafcd05edc8de09e305c772d2b8,refs/merge-requests/74/head,gitlab:GitlabProject:1:485905167
-gitlab:GitlabPipeline:1:485932863,12fc3a42080bb98ca520817bd4fe0ca33c0bb279,refs/merge-requests/74/head,gitlab:GitlabProject:1:485932863
+gitlab:GitlabPipeline:1:457474837,b1b82852d48b516a18e56c5bab0ebf54b8f4ccfd,renovate/pin-dependencies,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:457474996,739ab912e6e1ed27cecd8a2d00bfd6fa52afd90a,renovate/jest-monorepo,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:457475160,44d127e0ab7dbc4bc259b55929c9d00b62fc3bf4,renovate/lodash-monorepo,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:457475337,10a6464b6bd2cf4b59b8ac37ce1466e013f5a20d,renovate/shx-0.x,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485811050,c791ea6949d6b4aadf79b15ba666cb690c6527ac,EE-7121,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485811059,c791ea6949d6b4aadf79b15ba666cb690c6527ac,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485813816,ecc7c0b2874c812ed882c9effbbda26e0abc7110,EE-7121,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485813830,ecc7c0b2874c812ed882c9effbbda26e0abc7110,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485814501,6a3346f8434cc65fbe3f7a80a0edec5b4014a733,EE-7121,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485814516,6a3346f8434cc65fbe3f7a80a0edec5b4014a733,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485814871,f731732471961fed061a01fa99631f69e861c4a2,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485817670,5b95c5aebce1eae6a1b95ecf6fbc870851455375,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485837602,a79e2aaeee9916338a11b665cb4201f24627f3f1,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485842553,9e0934cedcd83abab8d34c5bb0f597c7837a55f0,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485845850,b3b581940f6d4d3226db31e48eab2b3b73f4c6db,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485852752,82df93eeb72c71271dad5a85359f39661de899fe,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485865876,db45efd231dbaca1337bd29bc1f65f404754e6bd,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485877118,09f81b1b2d083411c0bfecd32d7728479b594503,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485905167,cce432655861efafcd05edc8de09e305c772d2b8,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
+gitlab:GitlabPipeline:1:485932863,12fc3a42080bb98ca520817bd4fe0ca33c0bb279,refs/merge-requests/74/head,gitlab:GitlabProject:1:12345678
diff --git a/plugins/gitlab/e2e/snapshot_tables/cicd_tasks.csv b/plugins/gitlab/e2e/snapshot_tables/cicd_tasks.csv
index afc60453..88ca2e80 100644
--- a/plugins/gitlab/e2e/snapshot_tables/cicd_tasks.csv
+++ b/plugins/gitlab/e2e/snapshot_tables/cicd_tasks.csv
@@ -1,68 +1,68 @@
 id,name,pipeline_id,result,status,type,duration_sec,started_date,finished_date,environment
-gitlab:GitlabJob:1:100,100,gitlab:GitlabJob:1:24,SUCCESS,DONE,CI/CD,2,2022-07-25T15:06:57.051+00:00,2022-07-25T15:06:59.885+00:00,
-gitlab:GitlabJob:1:101,101,gitlab:GitlabJob:1:25,SUCCESS,DONE,CI/CD,3,2022-07-25T15:13:37.206+00:00,2022-07-25T15:13:40.246+00:00,
-gitlab:GitlabJob:1:102,102,gitlab:GitlabJob:1:26,SUCCESS,DONE,CI/CD,2,2022-07-25T15:30:22.560+00:00,2022-07-25T15:30:25.315+00:00,
-gitlab:GitlabJob:1:103,103,gitlab:GitlabJob:1:27,SUCCESS,DONE,CI/CD,2,2022-07-25T15:30:55.671+00:00,2022-07-25T15:30:58.650+00:00,
-gitlab:GitlabJob:1:104,104,gitlab:GitlabJob:1:28,SUCCESS,DONE,CI/CD,2,2022-07-25T15:32:04.954+00:00,2022-07-25T15:32:07.726+00:00,
-gitlab:GitlabJob:1:105,105,gitlab:GitlabJob:1:28,FAILURE,DONE,CI/CD,3,2022-07-25T15:32:07.953+00:00,2022-07-25T15:32:11.077+00:00,
-gitlab:GitlabJob:1:106,106,gitlab:GitlabJob:1:29,SUCCESS,DONE,CI/CD,2,2022-07-25T15:33:26.382+00:00,2022-07-25T15:33:29.356+00:00,
-gitlab:GitlabJob:1:107,107,gitlab:GitlabJob:1:30,SUCCESS,DONE,CI/CD,2,2022-07-25T15:34:23.665+00:00,2022-07-25T15:34:26.392+00:00,
-gitlab:GitlabJob:1:108,108,gitlab:GitlabJob:1:31,SUCCESS,DONE,CI/CD,2,2022-07-25T15:35:11.707+00:00,2022-07-25T15:35:14.224+00:00,
-gitlab:GitlabJob:1:109,109,gitlab:GitlabJob:1:31,SUCCESS,DONE,CI/CD,3,2022-07-25T15:35:14.724+00:00,2022-07-25T15:35:17.828+00:00,
-gitlab:GitlabJob:1:110,110,gitlab:GitlabJob:1:32,SUCCESS,DONE,CI/CD,2,2022-07-25T15:36:18.097+00:00,2022-07-25T15:36:20.954+00:00,
-gitlab:GitlabJob:1:111,111,gitlab:GitlabJob:1:33,SUCCESS,DONE,CI/CD,3,2022-07-25T15:38:03.463+00:00,2022-07-25T15:38:06.467+00:00,
-gitlab:GitlabJob:1:112,112,gitlab:GitlabJob:1:34,SUCCESS,DONE,CI/CD,3,2022-07-25T21:19:14.509+00:00,2022-07-25T21:19:17.811+00:00,
-gitlab:GitlabJob:1:113,113,gitlab:GitlabJob:1:35,SUCCESS,DONE,CI/CD,5,2022-07-26T09:37:05.694+00:00,2022-07-26T09:37:10.873+00:00,
-gitlab:GitlabJob:1:114,114,gitlab:GitlabJob:1:36,SUCCESS,DONE,CI/CD,2,2022-07-26T09:37:38.057+00:00,2022-07-26T09:37:40.975+00:00,
-gitlab:GitlabJob:1:115,115,gitlab:GitlabJob:1:37,SUCCESS,DONE,CI/CD,3,2022-07-26T09:38:29.318+00:00,2022-07-26T09:38:32.970+00:00,
-gitlab:GitlabJob:1:116,116,gitlab:GitlabJob:1:38,SUCCESS,DONE,CI/CD,3,2022-07-26T21:19:13.888+00:00,2022-07-26T21:19:17.021+00:00,
-gitlab:GitlabJob:1:117,117,gitlab:GitlabJob:1:39,SUCCESS,DONE,CI/CD,3,2022-07-27T08:19:24.376+00:00,2022-07-27T08:19:28.159+00:00,
-gitlab:GitlabJob:1:118,118,gitlab:GitlabJob:1:40,SUCCESS,DONE,CI/CD,4,2022-07-27T21:19:32.288+00:00,2022-07-27T21:19:36.850+00:00,
-gitlab:GitlabJob:1:119,119,gitlab:GitlabJob:1:41,FAILURE,DONE,CI/CD,0,2022-07-28T21:19:24.257+00:00,2022-07-28T23:00:17.842+00:00,
-gitlab:GitlabJob:1:120,120,gitlab:GitlabJob:1:41,SUCCESS,DONE,CI/CD,56,2022-07-29T02:10:58.370+00:00,2022-07-29T02:11:55.170+00:00,
-gitlab:GitlabJob:1:121,121,gitlab:GitlabJob:1:42,FAILURE,DONE,CI/CD,0,2022-07-29T21:19:02.884+00:00,2022-07-29T23:00:24.840+00:00,
-gitlab:GitlabJob:1:122,122,gitlab:GitlabJob:1:43,FAILURE,DONE,CI/CD,0,2022-07-30T21:19:26.310+00:00,2022-07-30T23:00:25.126+00:00,
-gitlab:GitlabJob:1:123,123,gitlab:GitlabJob:1:44,FAILURE,DONE,CI/CD,0,2022-07-31T21:19:05.348+00:00,2022-07-31T23:00:29.135+00:00,
-gitlab:GitlabJob:1:124,124,gitlab:GitlabJob:1:45,FAILURE,DONE,CI/CD,0,2022-08-01T21:19:02.489+00:00,2022-08-01T23:00:22.874+00:00,
-gitlab:GitlabJob:1:125,125,gitlab:GitlabJob:1:46,FAILURE,DONE,CI/CD,0,2022-08-02T21:19:25.568+00:00,2022-08-02T23:00:23.221+00:00,
-gitlab:GitlabJob:1:126,126,gitlab:GitlabJob:1:47,FAILURE,DONE,CI/CD,0,2022-08-03T08:19:06.570+00:00,2022-08-03T10:00:05.573+00:00,
-gitlab:GitlabJob:1:127,127,gitlab:GitlabJob:1:48,FAILURE,DONE,CI/CD,0,2022-08-03T21:19:21.010+00:00,2022-08-03T23:00:06.114+00:00,
-gitlab:GitlabJob:1:128,128,gitlab:GitlabJob:1:49,FAILURE,DONE,CI/CD,0,2022-08-04T21:19:12.398+00:00,2022-08-04T23:00:25.717+00:00,
-gitlab:GitlabJob:1:129,129,gitlab:GitlabJob:1:50,FAILURE,DONE,CI/CD,0,2022-08-05T21:19:09.648+00:00,2022-08-05T23:00:18.441+00:00,
-gitlab:GitlabJob:1:130,130,gitlab:GitlabJob:1:51,FAILURE,DONE,CI/CD,0,2022-08-06T21:19:29.253+00:00,2022-08-06T23:00:04.246+00:00,
-gitlab:GitlabJob:1:131,131,gitlab:GitlabJob:1:52,FAILURE,DONE,CI/CD,0,2022-08-07T21:19:33.476+00:00,2022-08-07T23:00:01.350+00:00,
-gitlab:GitlabJob:1:132,132,gitlab:GitlabJob:1:53,FAILURE,DONE,CI/CD,0,2022-08-08T21:19:02.531+00:00,2022-08-08T23:00:30.138+00:00,
-gitlab:GitlabJob:1:133,133,gitlab:GitlabJob:1:54,FAILURE,DONE,CI/CD,0,2022-08-09T21:19:34.379+00:00,2022-08-09T23:00:15.331+00:00,
-gitlab:GitlabJob:1:134,134,gitlab:GitlabJob:1:55,FAILURE,DONE,CI/CD,0,2022-08-10T08:19:08.693+00:00,2022-08-10T10:00:10.203+00:00,
-gitlab:GitlabJob:1:135,135,gitlab:GitlabJob:1:56,FAILURE,DONE,CI/CD,0,2022-08-10T21:19:05.714+00:00,2022-08-10T23:00:41.546+00:00,
-gitlab:GitlabJob:1:136,136,gitlab:GitlabJob:1:57,FAILURE,DONE,CI/CD,0,2022-08-11T21:19:25.605+00:00,2022-08-11T23:00:08.674+00:00,
-gitlab:GitlabJob:1:137,137,gitlab:GitlabJob:1:58,FAILURE,DONE,CI/CD,0,2022-08-12T21:19:08.350+00:00,2022-08-12T23:00:03.492+00:00,
-gitlab:GitlabJob:1:138,138,gitlab:GitlabJob:1:59,FAILURE,DONE,CI/CD,0,2022-08-13T21:19:06.775+00:00,2022-08-13T23:00:06.728+00:00,
-gitlab:GitlabJob:1:139,139,gitlab:GitlabJob:1:60,FAILURE,DONE,CI/CD,0,2022-08-14T21:19:07.007+00:00,2022-08-14T23:00:22.581+00:00,
-gitlab:GitlabJob:1:140,140,gitlab:GitlabJob:1:61,FAILURE,DONE,CI/CD,0,2022-08-15T21:19:09.087+00:00,2022-08-15T23:00:31.590+00:00,
-gitlab:GitlabJob:1:141,141,gitlab:GitlabJob:1:62,FAILURE,DONE,CI/CD,0,2022-08-16T21:19:12.248+00:00,2022-08-16T23:00:16.800+00:00,
-gitlab:GitlabJob:1:142,142,gitlab:GitlabJob:1:63,FAILURE,DONE,CI/CD,0,2022-08-17T08:20:06.419+00:00,2022-08-17T10:00:36.594+00:00,
-gitlab:GitlabJob:1:143,143,gitlab:GitlabJob:1:64,FAILURE,DONE,CI/CD,0,2022-08-17T21:19:11.908+00:00,2022-08-17T23:00:23.915+00:00,
-gitlab:GitlabJob:1:144,144,gitlab:GitlabJob:1:65,FAILURE,DONE,CI/CD,0,2022-08-18T21:19:14.072+00:00,2022-08-18T23:00:26.546+00:00,
-gitlab:GitlabJob:1:145,145,gitlab:GitlabJob:1:66,FAILURE,DONE,CI/CD,0,2022-08-19T21:19:03.364+00:00,2022-08-19T23:00:19.772+00:00,
-gitlab:GitlabJob:1:146,146,gitlab:GitlabJob:1:67,FAILURE,DONE,CI/CD,0,2022-08-20T21:19:37.743+00:00,2022-08-20T23:00:09.418+00:00,
-gitlab:GitlabJob:1:147,147,gitlab:GitlabJob:1:68,FAILURE,DONE,CI/CD,0,2022-08-21T21:19:02.164+00:00,2022-08-21T23:00:18.538+00:00,
-gitlab:GitlabJob:1:148,148,gitlab:GitlabJob:1:69,FAILURE,DONE,CI/CD,0,2022-08-22T21:19:16.175+00:00,2022-08-22T23:00:08.653+00:00,
-gitlab:GitlabJob:1:149,149,gitlab:GitlabJob:1:70,FAILURE,DONE,CI/CD,0,2022-08-23T21:19:13.313+00:00,2022-08-23T23:00:20.712+00:00,
-gitlab:GitlabJob:1:150,150,gitlab:GitlabJob:1:71,FAILURE,DONE,CI/CD,0,2022-08-24T08:19:19.653+00:00,2022-08-24T10:00:04.660+00:00,
-gitlab:GitlabJob:1:151,151,gitlab:GitlabJob:1:72,FAILURE,DONE,CI/CD,0,2022-08-24T21:19:29.226+00:00,2022-08-24T23:00:14.036+00:00,
-gitlab:GitlabJob:1:152,152,gitlab:GitlabJob:1:73,FAILURE,DONE,CI/CD,0,2022-08-25T21:19:10.938+00:00,2022-08-25T23:00:08.594+00:00,
-gitlab:GitlabJob:1:86,86,gitlab:GitlabJob:1:16,FAILURE,DONE,CI/CD,0,2022-07-25T13:40:42.020+00:00,2022-07-25T13:40:42.892+00:00,
-gitlab:GitlabJob:1:87,87,gitlab:GitlabJob:1:16,FAILURE,DONE,CI/CD,0,2022-07-25T13:41:11.601+00:00,2022-07-25T13:41:11.932+00:00,
-gitlab:GitlabJob:1:88,88,gitlab:GitlabJob:1:17,FAILURE,DONE,CI/CD,0,2022-07-25T13:42:59.674+00:00,2022-07-25T13:42:59.998+00:00,
-gitlab:GitlabJob:1:89,89,gitlab:GitlabJob:1:17,ABORT,DONE,CI/CD,0,2022-07-25T13:46:15.482+00:00,2022-07-25T13:49:42.952+00:00,
-gitlab:GitlabJob:1:90,90,gitlab:GitlabJob:1:18,ABORT,DONE,CI/CD,0,2022-07-25T13:50:40.680+00:00,2022-07-25T14:19:03.023+00:00,
-gitlab:GitlabJob:1:91,91,gitlab:GitlabJob:1:18,FAILURE,DONE,CI/CD,2,2022-07-25T14:26:02.616+00:00,2022-07-25T14:26:05.480+00:00,
-gitlab:GitlabJob:1:92,92,gitlab:GitlabJob:1:18,FAILURE,DONE,CI/CD,1,2022-07-25T14:47:12.876+00:00,2022-07-25T14:47:14.295+00:00,
-gitlab:GitlabJob:1:93,93,gitlab:GitlabJob:1:19,FAILURE,DONE,CI/CD,1,2022-07-25T14:53:56.227+00:00,2022-07-25T14:53:57.910+00:00,
-gitlab:GitlabJob:1:94,94,gitlab:GitlabJob:1:20,FAILURE,DONE,CI/CD,1,2022-07-25T14:55:26.493+00:00,2022-07-25T14:55:28.331+00:00,
-gitlab:GitlabJob:1:95,95,gitlab:GitlabJob:1:21,FAILURE,DONE,CI/CD,1,2022-07-25T14:56:59.811+00:00,2022-07-25T14:57:01.498+00:00,
-gitlab:GitlabJob:1:96,96,gitlab:GitlabJob:1:21,FAILURE,DONE,CI/CD,5,2022-07-25T14:59:29.276+00:00,2022-07-25T14:59:34.282+00:00,
-gitlab:GitlabJob:1:97,97,gitlab:GitlabJob:1:22,SUCCESS,DONE,CI/CD,3,2022-07-25T15:00:43.749+00:00,2022-07-25T15:00:46.895+00:00,
-gitlab:GitlabJob:1:98,98,gitlab:GitlabJob:1:23,SUCCESS,DONE,CI/CD,2,2022-07-25T15:03:23.471+00:00,2022-07-25T15:03:26.432+00:00,
-gitlab:GitlabJob:1:99,99,gitlab:GitlabJob:1:24,SUCCESS,DONE,CI/CD,2,2022-07-25T15:06:54.037+00:00,2022-07-25T15:06:56.819+00:00,
+gitlab:GitlabJob:1:100,compile,gitlab:GitlabPipeline:1:24,SUCCESS,DONE,,2,2022-07-25T15:06:57.051+00:00,2022-07-25T15:06:59.885+00:00,
+gitlab:GitlabJob:1:101,format,gitlab:GitlabPipeline:1:25,SUCCESS,DONE,,3,2022-07-25T15:13:37.206+00:00,2022-07-25T15:13:40.246+00:00,
+gitlab:GitlabJob:1:102,format,gitlab:GitlabPipeline:1:26,SUCCESS,DONE,,2,2022-07-25T15:30:22.560+00:00,2022-07-25T15:30:25.315+00:00,
+gitlab:GitlabJob:1:103,format,gitlab:GitlabPipeline:1:27,SUCCESS,DONE,,2,2022-07-25T15:30:55.671+00:00,2022-07-25T15:30:58.650+00:00,
+gitlab:GitlabJob:1:104,format,gitlab:GitlabPipeline:1:28,SUCCESS,DONE,,2,2022-07-25T15:32:04.954+00:00,2022-07-25T15:32:07.726+00:00,
+gitlab:GitlabJob:1:105,compile,gitlab:GitlabPipeline:1:28,FAILURE,DONE,,3,2022-07-25T15:32:07.953+00:00,2022-07-25T15:32:11.077+00:00,
+gitlab:GitlabJob:1:106,format,gitlab:GitlabPipeline:1:29,SUCCESS,DONE,,2,2022-07-25T15:33:26.382+00:00,2022-07-25T15:33:29.356+00:00,
+gitlab:GitlabJob:1:107,format,gitlab:GitlabPipeline:1:30,SUCCESS,DONE,,2,2022-07-25T15:34:23.665+00:00,2022-07-25T15:34:26.392+00:00,
+gitlab:GitlabJob:1:108,format,gitlab:GitlabPipeline:1:31,SUCCESS,DONE,,2,2022-07-25T15:35:11.707+00:00,2022-07-25T15:35:14.224+00:00,
+gitlab:GitlabJob:1:109,compile,gitlab:GitlabPipeline:1:31,SUCCESS,DONE,,3,2022-07-25T15:35:14.724+00:00,2022-07-25T15:35:17.828+00:00,
+gitlab:GitlabJob:1:110,format,gitlab:GitlabPipeline:1:32,SUCCESS,DONE,,2,2022-07-25T15:36:18.097+00:00,2022-07-25T15:36:20.954+00:00,
+gitlab:GitlabJob:1:111,format,gitlab:GitlabPipeline:1:33,SUCCESS,DONE,,3,2022-07-25T15:38:03.463+00:00,2022-07-25T15:38:06.467+00:00,
+gitlab:GitlabJob:1:112,format,gitlab:GitlabPipeline:1:34,SUCCESS,DONE,,3,2022-07-25T21:19:14.509+00:00,2022-07-25T21:19:17.811+00:00,
+gitlab:GitlabJob:1:113,format,gitlab:GitlabPipeline:1:35,SUCCESS,DONE,,5,2022-07-26T09:37:05.694+00:00,2022-07-26T09:37:10.873+00:00,
+gitlab:GitlabJob:1:114,format,gitlab:GitlabPipeline:1:36,SUCCESS,DONE,,2,2022-07-26T09:37:38.057+00:00,2022-07-26T09:37:40.975+00:00,
+gitlab:GitlabJob:1:115,format,gitlab:GitlabPipeline:1:37,SUCCESS,DONE,,3,2022-07-26T09:38:29.318+00:00,2022-07-26T09:38:32.970+00:00,
+gitlab:GitlabJob:1:116,format,gitlab:GitlabPipeline:1:38,SUCCESS,DONE,,3,2022-07-26T21:19:13.888+00:00,2022-07-26T21:19:17.021+00:00,
+gitlab:GitlabJob:1:117,format,gitlab:GitlabPipeline:1:39,SUCCESS,DONE,,3,2022-07-27T08:19:24.376+00:00,2022-07-27T08:19:28.159+00:00,
+gitlab:GitlabJob:1:118,format,gitlab:GitlabPipeline:1:40,SUCCESS,DONE,,4,2022-07-27T21:19:32.288+00:00,2022-07-27T21:19:36.850+00:00,
+gitlab:GitlabJob:1:119,format,gitlab:GitlabPipeline:1:41,FAILURE,DONE,,0,2022-07-28T21:19:24.257+00:00,2022-07-28T23:00:17.842+00:00,
+gitlab:GitlabJob:1:120,format,gitlab:GitlabPipeline:1:41,SUCCESS,DONE,,56,2022-07-29T02:10:58.370+00:00,2022-07-29T02:11:55.170+00:00,
+gitlab:GitlabJob:1:121,format,gitlab:GitlabPipeline:1:42,FAILURE,DONE,,0,2022-07-29T21:19:02.884+00:00,2022-07-29T23:00:24.840+00:00,
+gitlab:GitlabJob:1:122,format,gitlab:GitlabPipeline:1:43,FAILURE,DONE,,0,2022-07-30T21:19:26.310+00:00,2022-07-30T23:00:25.126+00:00,
+gitlab:GitlabJob:1:123,format,gitlab:GitlabPipeline:1:44,FAILURE,DONE,,0,2022-07-31T21:19:05.348+00:00,2022-07-31T23:00:29.135+00:00,
+gitlab:GitlabJob:1:124,format,gitlab:GitlabPipeline:1:45,FAILURE,DONE,,0,2022-08-01T21:19:02.489+00:00,2022-08-01T23:00:22.874+00:00,
+gitlab:GitlabJob:1:125,format,gitlab:GitlabPipeline:1:46,FAILURE,DONE,,0,2022-08-02T21:19:25.568+00:00,2022-08-02T23:00:23.221+00:00,
+gitlab:GitlabJob:1:126,format,gitlab:GitlabPipeline:1:47,FAILURE,DONE,,0,2022-08-03T08:19:06.570+00:00,2022-08-03T10:00:05.573+00:00,
+gitlab:GitlabJob:1:127,format,gitlab:GitlabPipeline:1:48,FAILURE,DONE,,0,2022-08-03T21:19:21.010+00:00,2022-08-03T23:00:06.114+00:00,
+gitlab:GitlabJob:1:128,format,gitlab:GitlabPipeline:1:49,FAILURE,DONE,,0,2022-08-04T21:19:12.398+00:00,2022-08-04T23:00:25.717+00:00,
+gitlab:GitlabJob:1:129,format,gitlab:GitlabPipeline:1:50,FAILURE,DONE,,0,2022-08-05T21:19:09.648+00:00,2022-08-05T23:00:18.441+00:00,
+gitlab:GitlabJob:1:130,format,gitlab:GitlabPipeline:1:51,FAILURE,DONE,,0,2022-08-06T21:19:29.253+00:00,2022-08-06T23:00:04.246+00:00,
+gitlab:GitlabJob:1:131,format,gitlab:GitlabPipeline:1:52,FAILURE,DONE,,0,2022-08-07T21:19:33.476+00:00,2022-08-07T23:00:01.350+00:00,
+gitlab:GitlabJob:1:132,format,gitlab:GitlabPipeline:1:53,FAILURE,DONE,,0,2022-08-08T21:19:02.531+00:00,2022-08-08T23:00:30.138+00:00,
+gitlab:GitlabJob:1:133,format,gitlab:GitlabPipeline:1:54,FAILURE,DONE,,0,2022-08-09T21:19:34.379+00:00,2022-08-09T23:00:15.331+00:00,
+gitlab:GitlabJob:1:134,format,gitlab:GitlabPipeline:1:55,FAILURE,DONE,,0,2022-08-10T08:19:08.693+00:00,2022-08-10T10:00:10.203+00:00,
+gitlab:GitlabJob:1:135,format,gitlab:GitlabPipeline:1:56,FAILURE,DONE,,0,2022-08-10T21:19:05.714+00:00,2022-08-10T23:00:41.546+00:00,
+gitlab:GitlabJob:1:136,format,gitlab:GitlabPipeline:1:57,FAILURE,DONE,,0,2022-08-11T21:19:25.605+00:00,2022-08-11T23:00:08.674+00:00,
+gitlab:GitlabJob:1:137,format,gitlab:GitlabPipeline:1:58,FAILURE,DONE,,0,2022-08-12T21:19:08.350+00:00,2022-08-12T23:00:03.492+00:00,
+gitlab:GitlabJob:1:138,format,gitlab:GitlabPipeline:1:59,FAILURE,DONE,,0,2022-08-13T21:19:06.775+00:00,2022-08-13T23:00:06.728+00:00,
+gitlab:GitlabJob:1:139,format,gitlab:GitlabPipeline:1:60,FAILURE,DONE,,0,2022-08-14T21:19:07.007+00:00,2022-08-14T23:00:22.581+00:00,
+gitlab:GitlabJob:1:140,format,gitlab:GitlabPipeline:1:61,FAILURE,DONE,,0,2022-08-15T21:19:09.087+00:00,2022-08-15T23:00:31.590+00:00,
+gitlab:GitlabJob:1:141,format,gitlab:GitlabPipeline:1:62,FAILURE,DONE,,0,2022-08-16T21:19:12.248+00:00,2022-08-16T23:00:16.800+00:00,
+gitlab:GitlabJob:1:142,format,gitlab:GitlabPipeline:1:63,FAILURE,DONE,,0,2022-08-17T08:20:06.419+00:00,2022-08-17T10:00:36.594+00:00,
+gitlab:GitlabJob:1:143,format,gitlab:GitlabPipeline:1:64,FAILURE,DONE,,0,2022-08-17T21:19:11.908+00:00,2022-08-17T23:00:23.915+00:00,
+gitlab:GitlabJob:1:144,format,gitlab:GitlabPipeline:1:65,FAILURE,DONE,,0,2022-08-18T21:19:14.072+00:00,2022-08-18T23:00:26.546+00:00,
+gitlab:GitlabJob:1:145,format,gitlab:GitlabPipeline:1:66,FAILURE,DONE,,0,2022-08-19T21:19:03.364+00:00,2022-08-19T23:00:19.772+00:00,
+gitlab:GitlabJob:1:146,format,gitlab:GitlabPipeline:1:67,FAILURE,DONE,,0,2022-08-20T21:19:37.743+00:00,2022-08-20T23:00:09.418+00:00,
+gitlab:GitlabJob:1:147,format,gitlab:GitlabPipeline:1:68,FAILURE,DONE,,0,2022-08-21T21:19:02.164+00:00,2022-08-21T23:00:18.538+00:00,
+gitlab:GitlabJob:1:148,format,gitlab:GitlabPipeline:1:69,FAILURE,DONE,,0,2022-08-22T21:19:16.175+00:00,2022-08-22T23:00:08.653+00:00,
+gitlab:GitlabJob:1:149,format,gitlab:GitlabPipeline:1:70,FAILURE,DONE,,0,2022-08-23T21:19:13.313+00:00,2022-08-23T23:00:20.712+00:00,
+gitlab:GitlabJob:1:150,format,gitlab:GitlabPipeline:1:71,FAILURE,DONE,,0,2022-08-24T08:19:19.653+00:00,2022-08-24T10:00:04.660+00:00,
+gitlab:GitlabJob:1:151,format,gitlab:GitlabPipeline:1:72,FAILURE,DONE,,0,2022-08-24T21:19:29.226+00:00,2022-08-24T23:00:14.036+00:00,
+gitlab:GitlabJob:1:152,format,gitlab:GitlabPipeline:1:73,FAILURE,DONE,,0,2022-08-25T21:19:10.938+00:00,2022-08-25T23:00:08.594+00:00,
+gitlab:GitlabJob:1:86,format,gitlab:GitlabPipeline:1:16,FAILURE,DONE,,0,2022-07-25T13:40:42.020+00:00,2022-07-25T13:40:42.892+00:00,
+gitlab:GitlabJob:1:87,format,gitlab:GitlabPipeline:1:16,FAILURE,DONE,,0,2022-07-25T13:41:11.601+00:00,2022-07-25T13:41:11.932+00:00,
+gitlab:GitlabJob:1:88,format,gitlab:GitlabPipeline:1:17,FAILURE,DONE,,0,2022-07-25T13:42:59.674+00:00,2022-07-25T13:42:59.998+00:00,
+gitlab:GitlabJob:1:89,format,gitlab:GitlabPipeline:1:17,ABORT,DONE,,0,2022-07-25T13:46:15.482+00:00,2022-07-25T13:49:42.952+00:00,
+gitlab:GitlabJob:1:90,format,gitlab:GitlabPipeline:1:18,ABORT,DONE,,0,2022-07-25T13:50:40.680+00:00,2022-07-25T14:19:03.023+00:00,
+gitlab:GitlabJob:1:91,format,gitlab:GitlabPipeline:1:18,FAILURE,DONE,,2,2022-07-25T14:26:02.616+00:00,2022-07-25T14:26:05.480+00:00,
+gitlab:GitlabJob:1:92,format,gitlab:GitlabPipeline:1:18,FAILURE,DONE,,1,2022-07-25T14:47:12.876+00:00,2022-07-25T14:47:14.295+00:00,
+gitlab:GitlabJob:1:93,format,gitlab:GitlabPipeline:1:19,FAILURE,DONE,,1,2022-07-25T14:53:56.227+00:00,2022-07-25T14:53:57.910+00:00,
+gitlab:GitlabJob:1:94,format,gitlab:GitlabPipeline:1:20,FAILURE,DONE,,1,2022-07-25T14:55:26.493+00:00,2022-07-25T14:55:28.331+00:00,
+gitlab:GitlabJob:1:95,format,gitlab:GitlabPipeline:1:21,FAILURE,DONE,,1,2022-07-25T14:56:59.811+00:00,2022-07-25T14:57:01.498+00:00,
+gitlab:GitlabJob:1:96,format,gitlab:GitlabPipeline:1:21,FAILURE,DONE,,5,2022-07-25T14:59:29.276+00:00,2022-07-25T14:59:34.282+00:00,
+gitlab:GitlabJob:1:97,format,gitlab:GitlabPipeline:1:22,SUCCESS,DONE,,3,2022-07-25T15:00:43.749+00:00,2022-07-25T15:00:46.895+00:00,
+gitlab:GitlabJob:1:98,format,gitlab:GitlabPipeline:1:23,SUCCESS,DONE,,2,2022-07-25T15:03:23.471+00:00,2022-07-25T15:03:26.432+00:00,
+gitlab:GitlabJob:1:99,format,gitlab:GitlabPipeline:1:24,SUCCESS,DONE,,2,2022-07-25T15:06:54.037+00:00,2022-07-25T15:06:56.819+00:00,
diff --git a/plugins/gitlab/models/migrationscripts/20220906_fix_duration_to_float8.go b/plugins/gitlab/models/migrationscripts/20220906_fix_duration_to_float8.go
index e2009c78..778438c8 100644
--- a/plugins/gitlab/models/migrationscripts/20220906_fix_duration_to_float8.go
+++ b/plugins/gitlab/models/migrationscripts/20220906_fix_duration_to_float8.go
@@ -49,7 +49,7 @@ func (*fixDurationToFloat8) Up(ctx context.Context, db *gorm.DB) error {
 	if err != nil {
 		return err
 	}
-	batch, err := helper.NewBatchUpdate(api.BasicRes, reflect.TypeOf(&GitlabJob20220906{}), 500)
+	batch, err := helper.NewBatchSave(api.BasicRes, reflect.TypeOf(&GitlabJob20220906{}), 500)
 	if err != nil {
 		return errors.Default.Wrap(err, "error getting batch from table", errors.UserMessage("Internal Converter execution error"))
 	}
diff --git a/plugins/gitlab/tasks/commit_convertor.go b/plugins/gitlab/tasks/commit_convertor.go
index 0fd5d230..186b273d 100644
--- a/plugins/gitlab/tasks/commit_convertor.go
+++ b/plugins/gitlab/tasks/commit_convertor.go
@@ -32,8 +32,8 @@ import (
 var ConvertCommitsMeta = core.SubTaskMeta{
 	Name:             "convertApiCommits",
 	EntryPoint:       ConvertApiCommits,
-	EnabledByDefault: false,
-	Description:      "Add domain layer commit according to GitlabCommit",
+	EnabledByDefault: true,
+	Description:      "Update domain layer commit according to GitlabCommit",
 	DomainTypes:      []string{core.DOMAIN_TYPE_CODE},
 }
 
diff --git a/plugins/gitlab/tasks/job_convertor.go b/plugins/gitlab/tasks/job_convertor.go
index 72b3ab34..d1e06890 100644
--- a/plugins/gitlab/tasks/job_convertor.go
+++ b/plugins/gitlab/tasks/job_convertor.go
@@ -18,7 +18,6 @@ limitations under the License.
 package tasks
 
 import (
-	"fmt"
 	"github.com/apache/incubator-devlake/models/domainlayer"
 	"github.com/apache/incubator-devlake/models/domainlayer/devops"
 	"github.com/apache/incubator-devlake/models/domainlayer/didgen"
@@ -48,7 +47,7 @@ func ConvertJobs(taskCtx core.SubTaskContext) error {
 	defer cursor.Close()
 
 	jobIdGen := didgen.NewDomainIdGenerator(&gitlabModels.GitlabJob{})
-
+	pipelineIdGen := didgen.NewDomainIdGenerator(&gitlabModels.GitlabPipeline{})
 	converter, err := helper.NewDataConverter(helper.DataConverterArgs{
 		InputRowType: reflect.TypeOf(gitlabModels.GitlabJob{}),
 		Input:        cursor,
@@ -73,18 +72,19 @@ func ConvertJobs(taskCtx core.SubTaskContext) error {
 					Id: jobIdGen.Generate(data.Options.ConnectionId, gitlabJob.GitlabId),
 				},
 
-				Name:       fmt.Sprintf("%d", gitlabJob.GitlabId),
-				PipelineId: jobIdGen.Generate(data.Options.ConnectionId, gitlabJob.PipelineId),
+				Name:       gitlabJob.Name,
+				PipelineId: pipelineIdGen.Generate(data.Options.ConnectionId, gitlabJob.PipelineId),
 				Result: devops.GetResult(&devops.ResultRule{
 					Failed:  []string{"failed"},
 					Abort:   []string{"canceled", "skipped"},
+					Manual:  []string{"manual"},
 					Default: devops.SUCCESS,
 				}, gitlabJob.Status),
 				Status: devops.GetStatus(&devops.StatusRule{
-					InProgress: []string{"created", "waiting_for_resource", "preparing", "pending", "running", "manual", "scheduled"},
+					InProgress: []string{"created", "waiting_for_resource", "preparing", "pending", "running", "scheduled"},
+					Manual:     []string{"manual"},
 					Default:    devops.DONE,
 				}, gitlabJob.Status),
-				Type: "CI/CD",
 
 				DurationSec:  uint64(gitlabJob.Duration),
 				StartedDate:  *startedAt,
diff --git a/plugins/gitlab/tasks/pipeline_project_convertor.go b/plugins/gitlab/tasks/pipeline_project_convertor.go
index 005d71cf..b826252b 100644
--- a/plugins/gitlab/tasks/pipeline_project_convertor.go
+++ b/plugins/gitlab/tasks/pipeline_project_convertor.go
@@ -69,7 +69,7 @@ func ConvertPipelineProjects(taskCtx core.SubTaskContext) error {
 				CommitSha: gitlabPipelineProject.Sha,
 				Branch:    gitlabPipelineProject.Ref,
 				Repo: didgen.NewDomainIdGenerator(&gitlabModels.GitlabProject{}).
-					Generate(gitlabPipelineProject.ConnectionId, gitlabPipelineProject.PipelineId),
+					Generate(gitlabPipelineProject.ConnectionId, gitlabPipelineProject.ProjectId),
 			}
 
 			return []interface{}{
diff --git a/plugins/helper/batch_save.go b/plugins/helper/batch_save.go
index bb37e19c..6aa9840d 100644
--- a/plugins/helper/batch_save.go
+++ b/plugins/helper/batch_save.go
@@ -19,7 +19,6 @@ package helper
 
 import (
 	"fmt"
-	"github.com/apache/incubator-devlake/errors"
 	"reflect"
 	"strings"
 
@@ -27,8 +26,8 @@ import (
 	"github.com/apache/incubator-devlake/plugins/core/dal"
 )
 
-// BatchShared is the base of BatchSave&BatchUpdate
-type BatchShared struct {
+// BatchSave performs mulitple records persistence of a specific type in one sql query to improve the performance
+type BatchSave struct {
 	basicRes core.BasicRes
 	log      core.Logger
 	db       dal.Dal
@@ -42,30 +41,20 @@ type BatchShared struct {
 	primaryKey []reflect.StructField
 }
 
-// BatchSave performs mulitple records persistence of a specific type in one sql query to improve the performance
-type BatchSave struct {
-	*BatchShared
-}
-
-// BatchUpdate will update records by batch
-type BatchUpdate struct {
-	*BatchShared
-}
-
-// NewBatchShared creates a new NewBatchShared instance to used in BatchSave&BatchUpdate
-func NewBatchShared(basicRes core.BasicRes, slotType reflect.Type, size int) (*BatchShared, error) {
+// NewBatchSave creates a new BatchSave instance
+func NewBatchSave(basicRes core.BasicRes, slotType reflect.Type, size int) (*BatchSave, error) {
 	if slotType.Kind() != reflect.Ptr {
-		return nil, errors.Default.New("slotType must be a pointer")
+		return nil, fmt.Errorf("slotType must be a pointer")
 	}
 	db := basicRes.GetDal()
 	primaryKey := db.GetPrimaryKeyFields(slotType)
 	// check if it have primaryKey
 	if len(primaryKey) == 0 {
-		return nil, errors.Default.New(fmt.Sprintf("%s no primary key", slotType.String()))
+		return nil, fmt.Errorf("%s no primary key", slotType.String())
 	}
 
 	log := basicRes.GetLogger().Nested(slotType.String())
-	batchShared := &BatchShared{
+	return &BatchSave{
 		basicRes:   basicRes,
 		log:        log,
 		db:         db,
@@ -74,65 +63,17 @@ func NewBatchShared(basicRes core.BasicRes, slotType reflect.Type, size int) (*B
 		size:       size,
 		valueIndex: make(map[string]int),
 		primaryKey: primaryKey,
-	}
-	return batchShared, nil
-}
-
-// NewBatchSave creates a new BatchSave instance
-func NewBatchSave(basicRes core.BasicRes, slotType reflect.Type, size int) (*BatchSave, error) {
-	batchShared, err := NewBatchShared(basicRes, slotType, size)
-	if err != nil {
-		return nil, err
-	}
-	return &BatchSave{batchShared}, nil
-}
-
-// NewBatchUpdate creates a new BatchUpdate instance
-func NewBatchUpdate(basicRes core.BasicRes, slotType reflect.Type, size int) (*BatchUpdate, error) {
-	batchShared, err := NewBatchShared(basicRes, slotType, size)
-	if err != nil {
-		return nil, err
-	}
-	return &BatchUpdate{batchShared}, nil
+	}, nil
 }
 
 // Add record to cache. BatchSave would flush them into Database when cache is max out
 func (c *BatchSave) Add(slot interface{}) error {
-	err := c.prepareForFlush(slot)
-	if err != nil {
-		return err
-	}
-	// flush out into database if max outed
-	if c.current == c.size {
-		return c.Flush()
-	} else if c.current%100 == 0 {
-		c.log.Debug("batch save current: %d", c.current)
-	}
-	return nil
-}
-
-// Add record to cache. BatchUpdate would flush them into Database when cache is max out
-func (c *BatchUpdate) Add(slot interface{}) error {
-	err := c.prepareForFlush(slot)
-	if err != nil {
-		return err
-	}
-	// flush out into database if max outed
-	if c.current == c.size {
-		return c.Flush()
-	} else if c.current%100 == 0 {
-		c.log.Debug("batch save current: %d", c.current)
-	}
-	return nil
-}
-
-func (c *BatchShared) prepareForFlush(slot interface{}) error {
 	// type checking
 	if reflect.TypeOf(slot) != c.slotType {
-		return errors.Default.New("sub cache type mismatched")
+		return fmt.Errorf("sub cache type mismatched")
 	}
 	if reflect.ValueOf(slot).Kind() != reflect.Ptr {
-		return errors.Default.New("slot is not a pointer")
+		return fmt.Errorf("slot is not a pointer")
 	}
 	// deduplication
 	key := getKeyValue(slot, c.primaryKey)
@@ -147,6 +88,12 @@ func (c *BatchShared) prepareForFlush(slot interface{}) error {
 	}
 	c.slots.Index(c.current).Set(reflect.ValueOf(slot))
 	c.current++
+	// flush out into database if max outed
+	if c.current == c.size {
+		return c.Flush()
+	} else if c.current%100 == 0 {
+		c.log.Debug("batch save current: %d", c.current)
+	}
 	return nil
 }
 
@@ -162,18 +109,6 @@ func (c *BatchSave) Flush() error {
 	return nil
 }
 
-// Flush update cached records into database
-func (c *BatchUpdate) Flush() error {
-	err := c.db.UpdateColumns(c.slots.Slice(0, c.current).Interface())
-	if err != nil {
-		return err
-	}
-	c.log.Debug("batch save flush total %d records to database", c.current)
-	c.current = 0
-	c.valueIndex = make(map[string]int)
-	return nil
-}
-
 // Close would flash the cache and release resources
 func (c *BatchSave) Close() error {
 	if c.current > 0 {
@@ -182,14 +117,6 @@ func (c *BatchSave) Close() error {
 	return nil
 }
 
-// Close would flash the cache and release resources
-func (c *BatchUpdate) Close() error {
-	if c.current > 0 {
-		return c.Flush()
-	}
-	return nil
-}
-
 func getKeyValue(iface interface{}, primaryKey []reflect.StructField) string {
 	var ss []string
 	ifv := reflect.ValueOf(iface)
diff --git a/plugins/helper/batch_save_divider.go b/plugins/helper/batch_save_divider.go
index 0555c986..6ec3c626 100644
--- a/plugins/helper/batch_save_divider.go
+++ b/plugins/helper/batch_save_divider.go
@@ -19,7 +19,6 @@ package helper
 
 import (
 	"fmt"
-	"github.com/apache/incubator-devlake/errors"
 	"reflect"
 
 	"github.com/apache/incubator-devlake/models/common"
@@ -27,65 +26,30 @@ import (
 	"github.com/apache/incubator-devlake/plugins/core/dal"
 )
 
-// BatchDivider is base struct of BatchSaveDivider&BatchUpdateDivider
-type BatchDivider struct {
+// BatchSaveDivider creates and caches BatchSave, this is helpful when dealing with massive amount of data records
+// with arbitrary types.
+type BatchSaveDivider struct {
 	basicRes  core.BasicRes
+	log       core.Logger
 	db        dal.Dal
+	batches   map[reflect.Type]*BatchSave
 	batchSize int
 	table     string
 	params    string
 }
 
-// BatchSaveDivider creates and caches BatchSave, this is helpful when dealing with massive amount of data records
-// with arbitrary types.
-type BatchSaveDivider struct {
-	*BatchDivider
-	log     core.Logger
-	batches map[reflect.Type]*BatchSave
-}
-
-// BatchUpdateDivider creates and caches BatchUpdate, this is helpful when dealing with massive amount of data records
-// with arbitrary types.
-type BatchUpdateDivider struct {
-	*BatchDivider
-	log     core.Logger
-	batches map[reflect.Type]*BatchUpdate
-}
-
 // NewBatchSaveDivider create a new BatchInsertDivider instance
 func NewBatchSaveDivider(basicRes core.BasicRes, batchSize int, table string, params string) *BatchSaveDivider {
 	log := basicRes.GetLogger().Nested("batch divider")
-	batchDivider := &BatchDivider{
+	return &BatchSaveDivider{
 		basicRes:  basicRes,
+		log:       log,
 		db:        basicRes.GetDal(),
+		batches:   make(map[reflect.Type]*BatchSave),
 		batchSize: batchSize,
 		table:     table,
 		params:    params,
 	}
-	batchSaveDivider := &BatchSaveDivider{
-		log:     log,
-		batches: make(map[reflect.Type]*BatchSave),
-	}
-	batchSaveDivider.BatchDivider = batchDivider
-	return batchSaveDivider
-}
-
-// NewBatchUpdateDivider create a new BatchInsertDivider instance
-func NewBatchUpdateDivider(basicRes core.BasicRes, batchSize int, table string, params string) *BatchUpdateDivider {
-	log := basicRes.GetLogger().Nested("batch update divider")
-	batchDivider := &BatchDivider{
-		basicRes:  basicRes,
-		db:        basicRes.GetDal(),
-		batchSize: batchSize,
-		table:     table,
-		params:    params,
-	}
-	batchUpdateDivider := &BatchUpdateDivider{
-		log:     log,
-		batches: make(map[reflect.Type]*BatchUpdate),
-	}
-	batchUpdateDivider.BatchDivider = batchDivider
-	return batchUpdateDivider
 }
 
 // ForType returns a `BatchSave` instance for specific type
@@ -107,7 +71,7 @@ func (d *BatchSaveDivider) ForType(rowType reflect.Type) (*BatchSave, error) {
 		// check if rowType had RawDataOrigin embeded
 		field, hasField := rowElemType.FieldByName("RawDataOrigin")
 		if !hasField || field.Type != reflect.TypeOf(common.RawDataOrigin{}) {
-			return nil, errors.Default.New(fmt.Sprintf("type %s must have RawDataOrigin embeded", rowElemType.Name()))
+			return nil, fmt.Errorf("type %s must have RawDataOrigin embeded", rowElemType.Name())
 		}
 		// all good, delete outdated records before we insertion
 		d.log.Debug("deleting outdate records for %s", rowElemType.Name())
@@ -132,30 +96,3 @@ func (d *BatchSaveDivider) Close() error {
 	}
 	return nil
 }
-
-// ForType returns a `BatchUpdate` instance for specific type
-func (d *BatchUpdateDivider) ForType(rowType reflect.Type) (*BatchUpdate, error) {
-	// get the cache for the specific type
-	batch := d.batches[rowType]
-	var err error
-	// create one if not exists
-	if batch == nil {
-		batch, err = NewBatchUpdate(d.basicRes, rowType, d.batchSize)
-		if err != nil {
-			return nil, err
-		}
-		d.batches[rowType] = batch
-	}
-	return batch, nil
-}
-
-// Close all batches so the rest records get saved into db
-func (d *BatchUpdateDivider) Close() error {
-	for _, batch := range d.batches {
-		err := batch.Close()
-		if err != nil {
-			return err
-		}
-	}
-	return nil
-}
diff --git a/plugins/helper/data_enricher.go b/plugins/helper/data_enricher.go
deleted file mode 100644
index e2f37d70..00000000
--- a/plugins/helper/data_enricher.go
+++ /dev/null
@@ -1,129 +0,0 @@
-/*
-Licensed to the Apache Software Foundation (ASF) under one or more
-contributor license agreements.  See the NOTICE file distributed with
-this work for additional information regarding copyright ownership.
-The ASF licenses this file to You under the Apache License, Version 2.0
-(the "License"); you may not use this file except in compliance with
-the License.  You may obtain a copy of the License at
-
-    http://www.apache.org/licenses/LICENSE-2.0
-
-Unless required by applicable law or agreed to in writing, software
-distributed under the License is distributed on an "AS IS" BASIS,
-WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
-See the License for the specific language governing permissions and
-limitations under the License.
-*/
-
-package helper
-
-import (
-	"database/sql"
-	"github.com/apache/incubator-devlake/errors"
-	"reflect"
-
-	"github.com/apache/incubator-devlake/plugins/core"
-)
-
-// DataEnrichHandler Accept row from source cursor, return list of entities that need to be stored
-type DataEnrichHandler func(row interface{}) ([]interface{}, error)
-
-// DataEnricherArgs includes the arguments about DataEnricher.
-// This will be used in Creating a DataEnricher.
-//
-//	DataEnricherArgs {
-//				InputRowType: 		type of inputRow ,
-//				Input:        		dal cursor,
-//				RawDataSubTaskArgs: args about raw data task
-//				Enrich: 			main function including conversion logic
-//				BatchSize: 			batch size
-type DataEnricherArgs struct {
-	RawDataSubTaskArgs
-	// Domain layer entity Id prefix, i.e. `jira:JiraIssue:1`, `github:GithubIssue`
-	InputRowType reflect.Type
-	Input        *sql.Rows
-	Enrich       DataEnrichHandler
-	BatchSize    int
-}
-
-// DataEnricher helps you convert Data from Tool Layer Tables to Domain Layer Tables
-// It reads rows from specified Iterator, and feed it into `Enricher` handler
-// you can return arbitrary domain layer entities from this handler, ApiEnricher would
-// first delete old data by their RawDataOrigin information, and then perform a
-// batch save operation for you.
-type DataEnricher struct {
-	*RawDataSubTask
-	args *DataEnricherArgs
-}
-
-// NewDataEnricher function helps you create a DataEnricher using DataEnricherArgs.
-// You can see the usage in plugins/github/tasks/pr_issue_convertor.go or other convertor file.
-func NewDataEnricher(args DataEnricherArgs) (*DataEnricher, error) {
-	rawDataSubTask, err := newRawDataSubTask(args.RawDataSubTaskArgs)
-	if err != nil {
-		return nil, err
-	}
-	// process args
-	if args.BatchSize == 0 {
-		args.BatchSize = 500
-	}
-	return &DataEnricher{
-		RawDataSubTask: rawDataSubTask,
-		args:           &args,
-	}, nil
-}
-
-// Execute function implements Subtask interface.
-// It loads data from Tool Layer Tables using `Ctx.GetDal()`, convert Data using `converter.args.Enrich` handler
-// Then save data to Domain Layer Tables using BatchSaveDivider
-func (enricher *DataEnricher) Execute() error {
-	// load data from database
-	db := enricher.args.Ctx.GetDal()
-
-	divider := NewBatchUpdateDivider(enricher.args.Ctx, enricher.args.BatchSize, enricher.table, enricher.params)
-
-	// set progress
-	enricher.args.Ctx.SetProgress(0, -1)
-
-	cursor := enricher.args.Input
-	defer cursor.Close()
-	ctx := enricher.args.Ctx.GetContext()
-	// iterate all rows
-	for cursor.Next() {
-		select {
-		case <-ctx.Done():
-			return ctx.Err()
-		default:
-		}
-		inputRow := reflect.New(enricher.args.InputRowType).Interface()
-		err := db.Fetch(cursor, inputRow)
-		if err != nil {
-			return errors.Default.Wrap(err, "error fetching rows", errors.UserMessage("Internal Enricher execution error"))
-		}
-
-		results, err := enricher.args.Enrich(inputRow)
-		if err != nil {
-			return errors.Default.Wrap(err, "error calling Enricher plugin implementation", errors.UserMessage("Internal Enricher execution error"))
-		}
-
-		for _, result := range results {
-			// get the batch operator for the specific type
-			batch, err := divider.ForType(reflect.TypeOf(result))
-			if err != nil {
-				return errors.Default.Wrap(err, "error getting batch from result", errors.UserMessage("Internal Enricher execution error"))
-			}
-			// records get saved into db when slots were max outed
-			err = batch.Add(result)
-			if err != nil {
-				return errors.Default.Wrap(err, "error updating result to batch", errors.UserMessage("Internal Enricher execution error"))
-			}
-		}
-		enricher.args.Ctx.IncProgress(1)
-	}
-
-	// save the last batches
-	return divider.Close()
-}
-
-// Check if DataEnricher implements SubTask interface
-var _ core.SubTask = (*DataEnricher)(nil)